Class MergingWindowSet<W extends Window>
- java.lang.Object
-
- org.apache.flink.streaming.runtime.operators.windowing.MergingWindowSet<W>
-
- Type Parameters:
W
- The type ofWindow
that this set is keeping track of.
public class MergingWindowSet<W extends Window> extends Object
Utility for keeping track of mergingWindows
when using aMergingWindowAssigner
in aWindowOperator
.When merging windows, we keep one of the original windows as the state window, i.e. the window that is used as namespace to store the window elements. Elements from the state windows of merged windows must be merged into this one state window. We keep a mapping from in-flight window to state window that can be queried using
getStateWindow(Window)
.A new window can be added to the set of in-flight windows using
addWindow(Window, MergeFunction)
. This might merge other windows and the caller must react accordingly in theMergingWindowSet.MergeFunction.merge(Object, Collection, Object, Collection)
and adjust the outside view of windows and state.Windows can be removed from the set of windows using
retireWindow(Window)
.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static interface
MergingWindowSet.MergeFunction<W>
Callback foraddWindow(Window, MergeFunction)
.
-
Constructor Summary
Constructors Constructor Description MergingWindowSet(MergingWindowAssigner<?,W> windowAssigner, ListState<Tuple2<W,W>> state)
Restores aMergingWindowSet
from the given state.
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description W
addWindow(W newWindow, MergingWindowSet.MergeFunction<W> mergeFunction)
Adds a newWindow
to the set of in-flight windows.W
getStateWindow(W window)
Returns the state window for the given in-flightWindow
.void
persist()
Persist the updated mapping to the given state if the mapping changed since initialization.void
retireWindow(W window)
Removes the given window from the set of in-flight windows.String
toString()
-
-
-
Constructor Detail
-
MergingWindowSet
public MergingWindowSet(MergingWindowAssigner<?,W> windowAssigner, ListState<Tuple2<W,W>> state) throws Exception
Restores aMergingWindowSet
from the given state.- Throws:
Exception
-
-
Method Detail
-
persist
public void persist() throws Exception
Persist the updated mapping to the given state if the mapping changed since initialization.- Throws:
Exception
-
getStateWindow
public W getStateWindow(W window)
Returns the state window for the given in-flightWindow
. The state window is theWindow
in which we keep the actual state of a given in-flight window. Windows might expand but we keep to original state window for keeping the elements of the window to avoid costly state juggling.- Parameters:
window
- The window for which to get the state window.
-
retireWindow
public void retireWindow(W window)
Removes the given window from the set of in-flight windows.- Parameters:
window
- TheWindow
to remove.
-
addWindow
public W addWindow(W newWindow, MergingWindowSet.MergeFunction<W> mergeFunction) throws Exception
Adds a newWindow
to the set of in-flight windows. It might happen that this triggers merging of previously in-flight windows. In that case, the providedMergingWindowSet.MergeFunction
is called.This returns the window that is the representative of the added window after adding. This can either be the new window itself, if no merge occurred, or the newly merged window. Adding an element to a window or calling trigger functions should only happen on the returned representative. This way, we never have to deal with a new window that is immediately swallowed up by another window.
If the new window is merged, the
MergeFunction
callback arguments also don't contain the new window as part of the list of merged windows.- Parameters:
newWindow
- The newWindow
to add.mergeFunction
- The callback to be invoked in case a merge occurs.- Returns:
- The
Window
that new newWindow
ended up in. This can also be the newWindow
itself in case no merge occurred. - Throws:
Exception
-
-