IN
- The type of the incoming elements.ACC
- The type of elements stored in the window buffers.OUT
- The type of elements emitted by the WindowFunction
.W
- The type of Window
that the WindowAssigner
assigns.@Internal public class NonKeyedWindowOperator<IN,ACC,OUT,W extends Window> extends AbstractUdfStreamOperator<OUT,AllWindowFunction<ACC,OUT,W>> implements OneInputStreamOperator<IN,OUT>, Triggerable, InputTypeConfigurable
WindowOperator
,
Serialized FormModifier and Type | Class and Description |
---|---|
protected class |
NonKeyedWindowOperator.Context
The
Context is responsible for keeping track of the state of one pane. |
Modifier and Type | Field and Description |
---|---|
protected long |
currentWatermark
To keep track of the current watermark so that we can immediately fire if a trigger
registers an event time callback for a timestamp that lies in the past.
|
protected TimestampedCollector<OUT> |
timestampedCollector
This is given to the
WindowFunction for emitting elements with a given timestamp. |
protected Map<W,NonKeyedWindowOperator.Context> |
windows
The windows (panes) that are currently in-flight.
|
userFunction
chainingStrategy, output
Constructor and Description |
---|
NonKeyedWindowOperator(WindowAssigner<? super IN,W> windowAssigner,
TypeSerializer<W> windowSerializer,
WindowBufferFactory<? super IN,ACC,? extends WindowBuffer<IN,ACC>> windowBufferFactory,
AllWindowFunction<ACC,OUT,W> windowFunction,
Trigger<? super IN,? super W> trigger)
Creates a new
WindowOperator based on the given policies and user functions. |
Modifier and Type | Method and Description |
---|---|
void |
dispose()
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 void |
emitWindow(NonKeyedWindowOperator.Context context) |
Trigger<? super IN,? super W> |
getTrigger() |
WindowAssigner<? super IN,W> |
getWindowAssigner() |
WindowBufferFactory<? super IN,ACC,? extends WindowBuffer<IN,ACC>> |
getWindowBufferFactory() |
void |
open()
This method is called immediately before any elements are processed, it should contain the
operator's initialization logic.
|
void |
processElement(StreamRecord<IN> element)
Processes one element that arrived at this operator.
|
void |
processWatermark(Watermark mark)
Processes a
Watermark . |
void |
restoreState(StreamTaskState taskState,
long recoveryTimestamp)
Restores the operator state, if this operator's execution is recovering from a checkpoint.
|
void |
setInputType(TypeInformation<?> type,
ExecutionConfig executionConfig)
Method that is called on an
OutputFormat when it is passed to
the DataSet's output method. |
StreamTaskState |
snapshotOperatorState(long checkpointId,
long timestamp)
Called to draw a state snapshot from the operator.
|
void |
trigger(long time)
This method is invoked with the timestamp for which the trigger was scheduled.
|
close, getUserFunction, getUserFunctionParameters, notifyOfCompletedCheckpoint, setOutputType, setup
disableInputCopy, getChainingStrategy, getContainingTask, getExecutionConfig, getOperatorConfig, getPartitionedState, getPartitionedState, getRuntimeContext, getStateBackend, getUserCodeClassloader, isInputCopyingDisabled, registerTimer, setChainingStrategy, setKeyContext, setKeyContextElement1, setKeyContextElement2
clone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, wait
close, getChainingStrategy, isInputCopyingDisabled, notifyOfCompletedCheckpoint, setChainingStrategy, setKeyContextElement1, setKeyContextElement2, setup
protected transient TimestampedCollector<OUT> timestampedCollector
WindowFunction
for emitting elements with a given timestamp.protected transient long currentWatermark
protected transient Map<W extends Window,NonKeyedWindowOperator.Context> windows
WindowBuffer
and a TriggerContext
that stores the Trigger
for that pane.public NonKeyedWindowOperator(WindowAssigner<? super IN,W> windowAssigner, TypeSerializer<W> windowSerializer, WindowBufferFactory<? super IN,ACC,? extends WindowBuffer<IN,ACC>> windowBufferFactory, AllWindowFunction<ACC,OUT,W> windowFunction, Trigger<? super IN,? super W> trigger)
WindowOperator
based on the given policies and user functions.public final void setInputType(TypeInformation<?> type, ExecutionConfig executionConfig)
InputTypeConfigurable
OutputFormat
when it is passed to
the DataSet's output method. May be used to configures the output format based on the data type.setInputType
in interface InputTypeConfigurable
type
- The data type of the input.public final void open() throws Exception
AbstractStreamOperator
The default implementation does nothing.
open
in interface StreamOperator<OUT>
open
in class AbstractUdfStreamOperator<OUT,AllWindowFunction<ACC,OUT,W extends Window>>
Exception
- An exception in this method causes the operator to fail.public final void dispose()
AbstractStreamOperator
dispose
in interface StreamOperator<OUT>
dispose
in class AbstractUdfStreamOperator<OUT,AllWindowFunction<ACC,OUT,W extends Window>>
public final void processElement(StreamRecord<IN> element) throws Exception
OneInputStreamOperator
processElement
in interface OneInputStreamOperator<IN,OUT>
Exception
protected void emitWindow(NonKeyedWindowOperator.Context context) throws Exception
Exception
public final void processWatermark(Watermark mark) throws Exception
OneInputStreamOperator
Watermark
.
This method is guaranteed to not be called concurrently with other methods of the operator.processWatermark
in interface OneInputStreamOperator<IN,OUT>
Exception
Watermark
public final void trigger(long time) throws Exception
Triggerable
If the triggering is delayed for whatever reason (trigger timer was blocked, JVM stalled due to a garbage collection), the timestamp supplied to this function will still be the original timestamp for which the trigger was scheduled.
trigger
in interface Triggerable
time
- The timestamp for which the trigger event was scheduled.Exception
public StreamTaskState snapshotOperatorState(long checkpointId, long timestamp) throws Exception
StreamOperator
snapshotOperatorState
in interface StreamOperator<OUT>
snapshotOperatorState
in class AbstractUdfStreamOperator<OUT,AllWindowFunction<ACC,OUT,W extends Window>>
checkpointId
- The ID of the checkpoint.timestamp
- The timestamp of the checkpoint.Exception
- Forwards exceptions that occur while drawing snapshots from the operator
and the key/value state.public void restoreState(StreamTaskState taskState, long recoveryTimestamp) throws Exception
StreamOperator
This method is called after StreamOperator.setup(StreamTask, StreamConfig, Output)
and before StreamOperator.open()
.
restoreState
in interface StreamOperator<OUT>
restoreState
in class AbstractUdfStreamOperator<OUT,AllWindowFunction<ACC,OUT,W extends Window>>
taskState
- The state of operator that was snapshotted as part of checkpoint
from which the execution is restored.recoveryTimestamp
- Global recovery timestampException
- Exceptions during state restore should be forwarded, so that the system can
properly react to failed state restore and fail the execution attempt.public WindowAssigner<? super IN,W> getWindowAssigner()
public WindowBufferFactory<? super IN,ACC,? extends WindowBuffer<IN,ACC>> getWindowBufferFactory()
Copyright © 2014–2017 The Apache Software Foundation. All rights reserved.