Class WindowOperator<K,W extends Window>
- java.lang.Object
-
- org.apache.flink.streaming.api.operators.AbstractStreamOperator<RowData>
-
- org.apache.flink.table.runtime.operators.window.groupwindow.operator.WindowOperator<K,W>
-
- Type Parameters:
K
- The type of key returned by theKeySelector
.W
- The type ofWindow
that theWindowAssigner
assigns.
- All Implemented Interfaces:
Serializable
,CheckpointListener
,Input<RowData>
,KeyContext
,KeyContextHandler
,OneInputStreamOperator<RowData,RowData>
,StreamOperator<RowData>
,StreamOperatorStateHandler.CheckpointedStreamOperator
,Triggerable<K,W>
,YieldingOperator<RowData>
- Direct Known Subclasses:
AggregateWindowOperator
,TableAggregateWindowOperator
public abstract class WindowOperator<K,W extends Window> extends AbstractStreamOperator<RowData> implements OneInputStreamOperator<RowData,RowData>, Triggerable<K,W>
An operator that implements the logic for windowing based on aGroupWindowAssigner
andTrigger
.This is the base class for
AggregateWindowOperator
andTableAggregateWindowOperator
. The big difference betweenAggregateWindowOperator
andTableAggregateWindowOperator
isAggregateWindowOperator
emits only one result for each aggregate group, whileTableAggregateWindowOperator
can emit multi results for each aggregate group.When an element arrives it gets assigned a key using a
KeySelector
and it gets assigned to zero or more windows using aGroupWindowAssigner
. Based on this, the element is put into panes. A pane is the bucket of elements that have the same key and sameWindow
. An element can be in multiple panes if it was assigned to multiple windows by theWindowAssigner
.Each pane gets its own instance of the provided
Trigger
. This trigger determines when the contents of the pane should be processed to emit results. When a trigger fires, the givenNamespaceAggsHandleFunctionBase
is invoked to produce the results that are emitted for the pane to which theTrigger
belongs.The parameter types:
<IN>
: RowData<OUT>
: JoinedRowData(KEY, AGG_RESULT)<KEY>
: GenericRowData<AGG_RESULT>
: GenericRowData<ACC>
: GenericRowData- See Also:
- Serialized Form
-
-
Field Summary
Fields Modifier and Type Field Description protected TimestampedCollector<RowData>
collector
This is used for emitting elements with a given timestamp.protected InternalValueState<K,W,RowData>
previousState
protected boolean
produceUpdates
protected RecordCounter
recordCounter
Used to count the number of added and retracted input records.protected ZoneId
shiftTimeZone
The shift timezone of the window, if the proctime or rowtime type is TIMESTAMP_LTZ, the shift timezone is the timezone user configured in TableConfig, other cases the timezone is UTC which means never shift when assigning windows.protected NamespaceAggsHandleFunctionBase<W>
windowAggregator
protected InternalWindowProcessFunction<K,W>
windowFunction
-
Fields inherited from class org.apache.flink.streaming.api.operators.AbstractStreamOperator
config, lastRecordAttributes1, lastRecordAttributes2, latencyStats, LOG, metrics, output, processingTimeService, stateHandler, stateKeySelector1, stateKeySelector2, timeServiceManager
-
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description void
close()
This method is called at the very end of the operator's life, both in the case of a successful completion of the operation, and in the case of a failure and canceling.protected abstract void
compileGeneratedCode()
protected abstract void
emitWindowResult(W window)
Emits the window result of the given window.protected Counter
getNumLateRecordsDropped()
protected Gauge<Long>
getWatermarkLatency()
void
onEventTime(InternalTimer<K,W> timer)
Invoked when an event-time timer fires.void
onProcessingTime(InternalTimer<K,W> timer)
Invoked when a processing-time timer fires.void
open()
This method is called immediately before any elements are processed, it should contain the operator's initialization logic, e.g. state initialization.void
processElement(StreamRecord<RowData> record)
Processes one element that arrived on this input of theMultipleInputStreamOperator
.boolean
useSplittableTimers()
Can be overridden to disable splittable timers for this particular operator even if config option is enabled.-
Methods inherited from class org.apache.flink.streaming.api.operators.AbstractStreamOperator
finish, getContainingTask, getCurrentKey, getExecutionConfig, getInternalTimerService, getKeyedStateBackend, getKeyedStateStore, getMetricGroup, getOperatorConfig, getOperatorID, getOperatorName, getOperatorStateBackend, getOrCreateKeyedState, getPartitionedState, getPartitionedState, getProcessingTimeService, getRuntimeContext, getStateKeySelector1, getStateKeySelector2, getTimeServiceManager, getUserCodeClassloader, hasKeyContext1, hasKeyContext2, initializeState, initializeState, isUsingCustomRawKeyedState, notifyCheckpointAborted, notifyCheckpointComplete, prepareSnapshotPreBarrier, processLatencyMarker, processLatencyMarker1, processLatencyMarker2, processRecordAttributes, processRecordAttributes1, processRecordAttributes2, processWatermark, processWatermark1, processWatermark2, processWatermarkStatus, processWatermarkStatus1, processWatermarkStatus2, reportOrForwardLatencyMarker, setCurrentKey, setKeyContextElement1, setKeyContextElement2, setMailboxExecutor, setProcessingTimeService, setup, snapshotState, snapshotState
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
-
Methods inherited from interface org.apache.flink.api.common.state.CheckpointListener
notifyCheckpointAborted, notifyCheckpointComplete
-
Methods inherited from interface org.apache.flink.streaming.api.operators.Input
processLatencyMarker, processRecordAttributes, processWatermark, processWatermarkStatus
-
Methods inherited from interface org.apache.flink.streaming.api.operators.KeyContext
getCurrentKey, setCurrentKey
-
Methods inherited from interface org.apache.flink.streaming.api.operators.KeyContextHandler
hasKeyContext
-
Methods inherited from interface org.apache.flink.streaming.api.operators.OneInputStreamOperator
setKeyContextElement
-
Methods inherited from interface org.apache.flink.streaming.api.operators.StreamOperator
finish, getMetricGroup, getOperatorAttributes, getOperatorID, initializeState, prepareSnapshotPreBarrier, setKeyContextElement1, setKeyContextElement2, snapshotState
-
-
-
-
Field Detail
-
produceUpdates
protected final boolean produceUpdates
-
shiftTimeZone
protected final ZoneId shiftTimeZone
The shift timezone of the window, if the proctime or rowtime type is TIMESTAMP_LTZ, the shift timezone is the timezone user configured in TableConfig, other cases the timezone is UTC which means never shift when assigning windows.
-
recordCounter
protected final RecordCounter recordCounter
Used to count the number of added and retracted input records.
-
windowAggregator
protected NamespaceAggsHandleFunctionBase<W extends Window> windowAggregator
-
windowFunction
protected transient InternalWindowProcessFunction<K,W extends Window> windowFunction
-
collector
protected transient TimestampedCollector<RowData> collector
This is used for emitting elements with a given timestamp.
-
previousState
protected transient InternalValueState<K,W extends Window,RowData> previousState
-
-
Method Detail
-
useSplittableTimers
public boolean useSplittableTimers()
Description copied from class:AbstractStreamOperator
Can be overridden to disable splittable timers for this particular operator even if config option is enabled. By default, splittable timers are disabled.- Overrides:
useSplittableTimers
in classAbstractStreamOperator<RowData>
- Returns:
true
if splittable timers should be used (subject toStreamConfig.isUnalignedCheckpointsEnabled()
andStreamConfig.isUnalignedCheckpointsSplittableTimersEnabled()
.false
if splittable timers should never be used.
-
compileGeneratedCode
protected abstract void compileGeneratedCode()
-
open
public void open() throws Exception
Description copied from class:AbstractStreamOperator
This method is called immediately before any elements are processed, it should contain the operator's initialization logic, e.g. state initialization.The default implementation does nothing.
- Specified by:
open
in interfaceStreamOperator<K>
- Overrides:
open
in classAbstractStreamOperator<RowData>
- Throws:
Exception
- An exception in this method causes the operator to fail.
-
close
public void close() throws Exception
Description copied from interface:StreamOperator
This method is called at the very end of the operator's life, both in the case of a successful completion of the operation, and in the case of a failure and canceling.This method is expected to make a thorough effort to release all resources that the operator has acquired.
NOTE:It can not emit any records! If you need to emit records at the end of processing, do so in the
StreamOperator.finish()
method.- Specified by:
close
in interfaceStreamOperator<K>
- Overrides:
close
in classAbstractStreamOperator<RowData>
- Throws:
Exception
-
processElement
public void processElement(StreamRecord<RowData> record) throws Exception
Description copied from interface:Input
Processes one element that arrived on this input of theMultipleInputStreamOperator
. This method is guaranteed to not be called concurrently with other methods of the operator.- Specified by:
processElement
in interfaceInput<K>
- Throws:
Exception
-
onEventTime
public void onEventTime(InternalTimer<K,W> timer) throws Exception
Description copied from interface:Triggerable
Invoked when an event-time timer fires.- Specified by:
onEventTime
in interfaceTriggerable<K,W extends Window>
- Throws:
Exception
-
onProcessingTime
public void onProcessingTime(InternalTimer<K,W> timer) throws Exception
Description copied from interface:Triggerable
Invoked when a processing-time timer fires.- Specified by:
onProcessingTime
in interfaceTriggerable<K,W extends Window>
- Throws:
Exception
-
emitWindowResult
protected abstract void emitWindowResult(W window) throws Exception
Emits the window result of the given window.- Throws:
Exception
-
getNumLateRecordsDropped
protected Counter getNumLateRecordsDropped()
-
-