Class Trigger<T,W extends Window>
- java.lang.Object
-
- org.apache.flink.streaming.api.windowing.triggers.Trigger<T,W>
-
- Type Parameters:
T
- The type of elements on which thisTrigger
works.W
- The type ofWindows
on which thisTrigger
can operate.
- All Implemented Interfaces:
Serializable
- Direct Known Subclasses:
ContinuousEventTimeTrigger
,ContinuousProcessingTimeTrigger
,CountTrigger
,DeltaTrigger
,EventTimeTrigger
,GlobalWindows.EndOfStreamTrigger
,GlobalWindows.NeverTrigger
,ProcessingTimeoutTrigger
,ProcessingTimeTrigger
,PurgingTrigger
@PublicEvolving public abstract class Trigger<T,W extends Window> extends Object implements Serializable
ATrigger
determines when a pane of a window should be evaluated to emit the results for that part of the window.A pane is the bucket of elements that have the same key (assigned by the
KeySelector
) and sameWindow
. An element can be in multiple panes if it was assigned to multiple windows by theWindowAssigner
. These panes all have their own instance of theTrigger
.Triggers must not maintain state internally since they can be re-created or reused for different keys. All necessary state should be persisted using the state abstraction available on the
Trigger.TriggerContext
.When used with a
MergingWindowAssigner
theTrigger
must returntrue
fromcanMerge()
andonMerge(Window, OnMergeContext)
most be properly implemented.- See Also:
- Serialized Form
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static interface
Trigger.OnMergeContext
Extension ofTrigger.TriggerContext
that is given toonMerge(Window, OnMergeContext)
.static interface
Trigger.TriggerContext
A context object that is given toTrigger
methods to allow them to register timer callbacks and deal with state.
-
Constructor Summary
Constructors Constructor Description Trigger()
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description boolean
canMerge()
Returns true if this trigger supports merging of trigger state and can therefore be used with aMergingWindowAssigner
.abstract void
clear(W window, Trigger.TriggerContext ctx)
Clears any state that the trigger might still hold for the given window.abstract TriggerResult
onElement(T element, long timestamp, W window, Trigger.TriggerContext ctx)
Called for every element that gets added to a pane.abstract TriggerResult
onEventTime(long time, W window, Trigger.TriggerContext ctx)
Called when an event-time timer that was set using the trigger context fires.void
onMerge(W window, Trigger.OnMergeContext ctx)
Called when several windows have been merged into one window by theWindowAssigner
.abstract TriggerResult
onProcessingTime(long time, W window, Trigger.TriggerContext ctx)
Called when a processing-time timer that was set using the trigger context fires.
-
-
-
Method Detail
-
onElement
public abstract TriggerResult onElement(T element, long timestamp, W window, Trigger.TriggerContext ctx) throws Exception
Called for every element that gets added to a pane. The result of this will determine whether the pane is evaluated to emit results.- Parameters:
element
- The element that arrived.timestamp
- The timestamp of the element that arrived.window
- The window to which the element is being added.ctx
- A context object that can be used to register timer callbacks.- Throws:
Exception
-
onProcessingTime
public abstract TriggerResult onProcessingTime(long time, W window, Trigger.TriggerContext ctx) throws Exception
Called when a processing-time timer that was set using the trigger context fires.- Parameters:
time
- The timestamp at which the timer fired.window
- The window for which the timer fired.ctx
- A context object that can be used to register timer callbacks.- Throws:
Exception
-
onEventTime
public abstract TriggerResult onEventTime(long time, W window, Trigger.TriggerContext ctx) throws Exception
Called when an event-time timer that was set using the trigger context fires.- Parameters:
time
- The timestamp at which the timer fired.window
- The window for which the timer fired.ctx
- A context object that can be used to register timer callbacks.- Throws:
Exception
-
canMerge
public boolean canMerge()
Returns true if this trigger supports merging of trigger state and can therefore be used with aMergingWindowAssigner
.If this returns
true
you must properly implementonMerge(Window, OnMergeContext)
-
onMerge
public void onMerge(W window, Trigger.OnMergeContext ctx) throws Exception
Called when several windows have been merged into one window by theWindowAssigner
.- Parameters:
window
- The new window that results from the merge.ctx
- A context object that can be used to register timer callbacks and access state.- Throws:
Exception
-
clear
public abstract void clear(W window, Trigger.TriggerContext ctx) throws Exception
Clears any state that the trigger might still hold for the given window. This is called when a window is purged. Timers set usingTrigger.TriggerContext.registerEventTimeTimer(long)
andTrigger.TriggerContext.registerProcessingTimeTimer(long)
should be deleted here as well as state acquired usingTrigger.TriggerContext.getPartitionedState(StateDescriptor)
.- Throws:
Exception
-
-