T
- type of elements being windowedW
- BoundedWindow
subclass used to represent the windows used by this WindowFn
public abstract class WindowFn<T,W extends BoundedWindow> extends java.lang.Object implements java.io.Serializable, HasDisplayData
Window
transform used to assign elements into windows and to
determine how windows are merged. See Window
for more information on how WindowFn
s are used and for a library of predefined WindowFns
.
Users will generally want to use the predefined WindowFns
, but it is also
possible to create new subclasses.
To create a custom WindowFn
, inherit from this class and override all required
methods. If no merging is required, inherit from NonMergingWindowFn
instead. If no
merging is required and each element is assigned to a single window, inherit from PartitioningWindowFn
. Inheriting from the most specific subclass will enable more optimizations
in the runner.
Modifier and Type | Class and Description |
---|---|
class |
WindowFn.AssignContext
Information available when running
assignWindows(org.apache.beam.sdk.transforms.windowing.WindowFn<T, W>.AssignContext) . |
class |
WindowFn.MergeContext
Information available when running
mergeWindows(org.apache.beam.sdk.transforms.windowing.WindowFn<T, W>.MergeContext) . |
Constructor and Description |
---|
WindowFn() |
Modifier and Type | Method and Description |
---|---|
boolean |
assignsToOneWindow()
Returns true if this
WindowFn always assigns an element to exactly one window. |
abstract java.util.Collection<W> |
assignWindows(WindowFn.AssignContext c)
Given a timestamp and element, returns the set of windows into which it should be placed.
|
abstract WindowMappingFn<W> |
getDefaultWindowMappingFn()
Returns the default
WindowMappingFn to use to map main input windows to side input
windows. |
TypeDescriptor<W> |
getWindowTypeDescriptor()
Returns a
TypeDescriptor capturing what is known statically about the window type of
this WindowFn instance's most-derived class. |
abstract boolean |
isCompatible(WindowFn<?,?> other)
Deprecated.
please override verifyCompatibility to throw a useful error message; we will remove
isCompatible at version 3.0.0
|
boolean |
isNonMerging()
Returns true if this
WindowFn never needs to merge any windows. |
abstract void |
mergeWindows(WindowFn.MergeContext c)
Does whatever merging of windows is necessary.
|
void |
populateDisplayData(DisplayData.Builder builder)
Register display data for the given transform or component.
|
void |
verifyCompatibility(WindowFn<?,?> other)
Throw
IncompatibleWindowException if this WindowFn does not perform the same merging as
the given $WindowFn . |
abstract Coder<W> |
windowCoder()
Returns the
Coder used for serializing the windows used by this windowFn. |
public abstract java.util.Collection<W> assignWindows(WindowFn.AssignContext c) throws java.lang.Exception
java.lang.Exception
public abstract void mergeWindows(WindowFn.MergeContext c) throws java.lang.Exception
See MergeOverlappingIntervalWindows.mergeWindows(org.apache.beam.sdk.transforms.windowing.WindowFn<?, org.apache.beam.sdk.transforms.windowing.IntervalWindow>.MergeContext)
for an example of how to override
this method.
java.lang.Exception
@Deprecated public abstract boolean isCompatible(WindowFn<?,?> other)
WindowFn
.public void verifyCompatibility(WindowFn<?,?> other) throws IncompatibleWindowException
IncompatibleWindowException
if this WindowFn does not perform the same merging as
the given $WindowFn
.IncompatibleWindowException
- if compared WindowFns are not compatible.public abstract Coder<W> windowCoder()
Coder
used for serializing the windows used by this windowFn.public abstract WindowMappingFn<W> getDefaultWindowMappingFn()
WindowMappingFn
to use to map main input windows to side input
windows. This should accept arbitrary main input windows, and produce a BoundedWindow
that can be produced by this WindowFn
.public boolean isNonMerging()
WindowFn
never needs to merge any windows.public boolean assignsToOneWindow()
WindowFn
always assigns an element to exactly one window.
If this varies per-element, or cannot be determined, conservatively return false.
By default, returns false.
public TypeDescriptor<W> getWindowTypeDescriptor()
TypeDescriptor
capturing what is known statically about the window type of
this WindowFn
instance's most-derived class.
In the normal case of a concrete WindowFn
subclass with no generic type parameters
of its own (including anonymous inner classes), this will be a complete non-generic type.
public void populateDisplayData(DisplayData.Builder builder)
populateDisplayData(DisplayData.Builder)
is invoked by Pipeline runners to collect
display data via DisplayData.from(HasDisplayData)
. Implementations may call super.populateDisplayData(builder)
in order to register display data in the current namespace,
but should otherwise use subcomponent.populateDisplayData(builder)
to use the namespace
of the subcomponent.
By default, does not register any display data. Implementors may override this method to provide their own display data.
populateDisplayData
in interface HasDisplayData
builder
- The builder to populate with display data.HasDisplayData