Interface AsyncStateWindowBuffer
- All Known Implementing Classes:
AsyncStateRecordsWindowBuffer
-
Nested Class Summary
Nested ClassesModifier and TypeInterfaceDescriptionstatic interfaceA factory that creates aWindowBufferwith async state. -
Method Summary
Modifier and TypeMethodDescriptionorg.apache.flink.api.common.state.v2.StateFuture<Void>addElement(org.apache.flink.table.data.RowData dataKey, long window, org.apache.flink.table.data.RowData element) Adds an element with associated key into the buffer.org.apache.flink.api.common.state.v2.StateFuture<Void>advanceProgress(org.apache.flink.table.data.RowData currentKey, long progress) Advances the progress time, the progress time is watermark if working in event-time mode, or current processing time if working in processing-time mode.voidclose()Release resources allocated by this buffer.org.apache.flink.api.common.state.v2.StateFuture<Void>flush(org.apache.flink.table.data.RowData currentKey) Flushes all intermediate buffered data to the underlying backend or output stream.
-
Method Details
-
addElement
org.apache.flink.api.common.state.v2.StateFuture<Void> addElement(org.apache.flink.table.data.RowData dataKey, long window, org.apache.flink.table.data.RowData element) throws Exception Adds an element with associated key into the buffer. The buffer may temporarily buffer the element, or immediately write it to the stream.It may be that adding this element fills up an internal buffer and causes the buffer flushing of a batch of internally buffered elements.
- Parameters:
dataKey- the key associated with the elementelement- The element to add.- Throws:
Exception- Thrown, if the element cannot be added to the buffer, or if the flushing throws an exception.
-
advanceProgress
org.apache.flink.api.common.state.v2.StateFuture<Void> advanceProgress(@Nullable org.apache.flink.table.data.RowData currentKey, long progress) throws Exception Advances the progress time, the progress time is watermark if working in event-time mode, or current processing time if working in processing-time mode.This will potentially flush buffered data into states or to the output stream, because the watermark advancement may be in a very small step, but we don't need to flush buffered data for every watermark advancement.
Note: There may be multiple different keys within the buffer. When flushing them to the async state, only the async state request for the current key of the operator will be returned as a
StateFuture. Requests for async states for other keys will not be waited on.- Parameters:
currentKey- the current key when processing and is used to return the result of accessing async state associated with the same key. If it is null, it means that the returns of asynchronous state requests for all keys will not be awaited.progress- the current progress time- Returns:
- the future of the flush operation about current key if the current key is not null,
else a
StateFutureUtils.completedVoidFuture()will be returned. - Throws:
Exception
-
flush
org.apache.flink.api.common.state.v2.StateFuture<Void> flush(@Nullable org.apache.flink.table.data.RowData currentKey) throws Exception Flushes all intermediate buffered data to the underlying backend or output stream.Note: There may be multiple different keys within the buffer. When flushing them to the async state, only the async state request for the current key of the operator will be returned as a
StateFuture. Requests for async states for other keys will not be waited on.- Parameters:
currentKey- the current key when processing and is used to return the result of accessing async state associated with the same key. If it is null, it means that the returns of asynchronous state requests for all keys will not be awaited.- Returns:
- the future of the flush operation about current key if the current key is not null,
else a
StateFutureUtils.completedVoidFuture()will be returned. - Throws:
Exception- Thrown if the buffer cannot be flushed, or if the output stream throws an exception.
-
close
Release resources allocated by this buffer.- Throws:
Exception
-