Class SlidingWindows
- All Implemented Interfaces:
Serializable
,HasDisplayData
WindowFn
that windows values into possibly overlapping fixed-size timestamp-based
windows.
For example, in order to window data into 10 minute windows that update every minute:
PCollection<Integer> items = ...;
PCollection<Integer> windowedItems = items.apply(
Window.<Integer>into(SlidingWindows.of(Duration.standardMinutes(10))));
- See Also:
-
Nested Class Summary
Nested classes/interfaces inherited from class org.apache.beam.sdk.transforms.windowing.WindowFn
WindowFn.AssignContext, WindowFn.MergeContext
-
Method Summary
Modifier and TypeMethodDescriptionboolean
Returns true if thisWindowFn
always assigns an element to exactly one window.Given a timestamp and element, returns the set of windows into which it should be placed.assignWindows
(Instant timestamp) boolean
Returns a newSlidingWindows
with the original size, that assigns timestamps into half-open intervals of the form [N * period, N * period + size), where 0 is the epoch.Return aWindowMappingFn
that returns the earliest window that contains the end of the main-input window.getSize()
int
hashCode()
boolean
isCompatible
(WindowFn<?, ?> other) Returns whether this performs the same merging as the givenWindowFn
.static SlidingWindows
Assigns timestamps into half-open intervals of the form [N * period, N * period + size), where 0 is the epoch.void
populateDisplayData
(DisplayData.Builder builder) Register display data for the given transform or component.void
verifyCompatibility
(WindowFn<?, ?> other) ThrowIncompatibleWindowException
if this WindowFn does not perform the same merging as the given $WindowFn
.Returns theCoder
used for serializing the windows used by this windowFn.withOffset
(Duration offset) Assigns timestamps into half-open intervals of the form [N * period + offset, N * period + offset + size).Methods inherited from class org.apache.beam.sdk.transforms.windowing.NonMergingWindowFn
isNonMerging, mergeWindows
Methods inherited from class org.apache.beam.sdk.transforms.windowing.WindowFn
getWindowTypeDescriptor
-
Method Details
-
of
Assigns timestamps into half-open intervals of the form [N * period, N * period + size), where 0 is the epoch.If
every(org.joda.time.Duration)
is not called, the period defaults to the largest time unit smaller than the given duration. For example, specifying a size of 5 seconds will result in a default period of 1 second. -
every
Returns a newSlidingWindows
with the original size, that assigns timestamps into half-open intervals of the form [N * period, N * period + size), where 0 is the epoch. -
withOffset
Assigns timestamps into half-open intervals of the form [N * period + offset, N * period + offset + size).- Throws:
IllegalArgumentException
- if offset is not in [0, period)
-
windowCoder
Description copied from class:WindowFn
Returns theCoder
used for serializing the windows used by this windowFn.- Specified by:
windowCoder
in classWindowFn<Object,
IntervalWindow>
-
assignWindows
Description copied from class:WindowFn
Given a timestamp and element, returns the set of windows into which it should be placed.- Specified by:
assignWindows
in classWindowFn<Object,
IntervalWindow>
-
assignWindows
-
getDefaultWindowMappingFn
Return aWindowMappingFn
that returns the earliest window that contains the end of the main-input window.- Specified by:
getDefaultWindowMappingFn
in classWindowFn<Object,
IntervalWindow>
-
isCompatible
Description copied from class:WindowFn
Returns whether this performs the same merging as the givenWindowFn
.- Specified by:
isCompatible
in classWindowFn<Object,
IntervalWindow>
-
assignsToOneWindow
public boolean assignsToOneWindow()Description copied from class:WindowFn
Returns true if thisWindowFn
always assigns an element to exactly one window.If this varies per-element, or cannot be determined, conservatively return false.
By default, returns false.
- Overrides:
assignsToOneWindow
in classWindowFn<Object,
IntervalWindow>
-
verifyCompatibility
Description copied from class:WindowFn
ThrowIncompatibleWindowException
if this WindowFn does not perform the same merging as the given $WindowFn
.- Overrides:
verifyCompatibility
in classWindowFn<Object,
IntervalWindow> - Throws:
IncompatibleWindowException
- if compared WindowFns are not compatible.
-
populateDisplayData
Description copied from class:WindowFn
Register display data for the given transform or component.populateDisplayData(DisplayData.Builder)
is invoked by Pipeline runners to collect display data viaDisplayData.from(HasDisplayData)
. Implementations may callsuper.populateDisplayData(builder)
in order to register display data in the current namespace, but should otherwise usesubcomponent.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.
- Specified by:
populateDisplayData
in interfaceHasDisplayData
- Overrides:
populateDisplayData
in classWindowFn<Object,
IntervalWindow> - Parameters:
builder
- The builder to populate with display data.- See Also:
-
getPeriod
-
getSize
-
getOffset
-
equals
-
hashCode
public int hashCode()
-