T
- The type of elements on which this Trigger
works.W
- The type of Windows
on which this Trigger
can operate.@PublicEvolving public abstract class Trigger<T,W extends Window> extends Object implements Serializable
Trigger
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 same Window
. An element can
be in multiple panes if it was assigned to multiple windows by the
WindowAssigner
. These panes all
have their own instance of the Trigger
.
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
the Trigger
must return true
from canMerge()
and
onMerge(Window, OnMergeContext)
most be properly implemented.
Modifier and Type | Class and Description |
---|---|
static interface |
Trigger.OnMergeContext
Extension of
Trigger.TriggerContext that is given to
onMerge(Window, OnMergeContext) . |
static interface |
Trigger.TriggerContext
A context object that is given to
Trigger methods to allow them to register timer
callbacks and deal with state. |
Constructor and Description |
---|
Trigger() |
Modifier and Type | Method and Description |
---|---|
boolean |
canMerge()
Returns true if this trigger supports merging of trigger state and can therefore
be used with a
MergingWindowAssigner . |
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 the
WindowAssigner . |
abstract TriggerResult |
onProcessingTime(long time,
W window,
Trigger.TriggerContext ctx)
Called when a processing-time timer that was set using the trigger context fires.
|
public abstract TriggerResult onElement(T element, long timestamp, W window, Trigger.TriggerContext ctx) throws Exception
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.Exception
public abstract TriggerResult onProcessingTime(long time, W window, Trigger.TriggerContext ctx) throws Exception
Note: This method is not called in case the window does not contain any elements. Thus,
if you return PURGE
from a trigger method and you expect to do cleanup in a future
invocation of a timer callback it might be wise to clean any state that you would clean
in the timer callback.
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.Exception
public abstract TriggerResult onEventTime(long time, W window, Trigger.TriggerContext ctx) throws Exception
Note: This method is not called in case the window does not contain any elements. Thus,
if you return PURGE
from a trigger method and you expect to do cleanup in a future
invocation of a timer callback it might be wise to clean any state that you would clean
in the timer callback.
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.Exception
public boolean canMerge()
MergingWindowAssigner
.
If this returns true
you must properly implement
onMerge(Window, OnMergeContext)
public void onMerge(W window, Trigger.OnMergeContext ctx) throws Exception
WindowAssigner
.window
- The new window that results from the merge.ctx
- A context object that can be used to register timer callbacks and access state.Exception
public abstract void clear(W window, Trigger.TriggerContext ctx) throws Exception
Trigger.TriggerContext.registerEventTimeTimer(long)
and Trigger.TriggerContext.registerProcessingTimeTimer(long)
should be deleted here as
well as state acquired using Trigger.TriggerContext.getPartitionedState(StateDescriptor)
.Exception
Copyright © 2014–2019 The Apache Software Foundation. All rights reserved.