Class SlidingEventTimeWindows
- java.lang.Object
-
- org.apache.flink.streaming.api.windowing.assigners.WindowAssigner<Object,TimeWindow>
-
- org.apache.flink.streaming.api.windowing.assigners.SlidingEventTimeWindows
-
- All Implemented Interfaces:
Serializable
@PublicEvolving public class SlidingEventTimeWindows extends WindowAssigner<Object,TimeWindow>
AWindowAssigner
that windows elements into sliding windows based on the timestamp of the elements. Windows can possibly overlap.For example, in order to window into windows of 1 minute, every 10 seconds:
DataStream<Tuple2<String, Integer>> in = ...; KeyedStream<Tuple2<String, Integer>, String> keyed = in.keyBy(...); WindowedStream<Tuple2<String, Integer>, String, TimeWindow> windowed = keyed.window(SlidingEventTimeWindows.of(Duration.ofMinutes(1), Duration.ofSeconds(10)));
- See Also:
- Serialized Form
-
-
Nested Class Summary
-
Nested classes/interfaces inherited from class org.apache.flink.streaming.api.windowing.assigners.WindowAssigner
WindowAssigner.WindowAssignerContext
-
-
Constructor Summary
Constructors Modifier Constructor Description protected
SlidingEventTimeWindows(long size, long slide, long offset)
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Modifier and Type Method Description Collection<TimeWindow>
assignWindows(Object element, long timestamp, WindowAssigner.WindowAssignerContext context)
Returns aCollection
of windows that should be assigned to the element.Trigger<Object,TimeWindow>
getDefaultTrigger()
Returns the default trigger associated with thisWindowAssigner
.long
getSize()
long
getSlide()
TypeSerializer<TimeWindow>
getWindowSerializer(ExecutionConfig executionConfig)
Returns aTypeSerializer
for serializing windows that are assigned by thisWindowAssigner
.boolean
isEventTime()
Returnstrue
if elements are assigned to windows based on event time,false
otherwise.static SlidingEventTimeWindows
of(Duration size, Duration slide)
Creates a newSlidingEventTimeWindows
WindowAssigner
that assigns elements to sliding time windows based on the element timestamp.static SlidingEventTimeWindows
of(Duration size, Duration slide, Duration offset)
Creates a newSlidingEventTimeWindows
WindowAssigner
that assigns elements to time windows based on the element timestamp and offset.String
toString()
-
-
-
Method Detail
-
assignWindows
public Collection<TimeWindow> assignWindows(Object element, long timestamp, WindowAssigner.WindowAssignerContext context)
Description copied from class:WindowAssigner
Returns aCollection
of windows that should be assigned to the element.- Specified by:
assignWindows
in classWindowAssigner<Object,TimeWindow>
- Parameters:
element
- The element to which windows should be assigned.timestamp
- The timestamp of the element.context
- TheWindowAssigner.WindowAssignerContext
in which the assigner operates.
-
getSize
public long getSize()
-
getSlide
public long getSlide()
-
getDefaultTrigger
public Trigger<Object,TimeWindow> getDefaultTrigger()
Description copied from class:WindowAssigner
Returns the default trigger associated with thisWindowAssigner
.1. If you override
getDefaultTrigger()
, thegetDefaultTrigger()
will be invoked and thegetDefaultTrigger(StreamExecutionEnvironment env)
won't be invoked. 2. If you don't overridegetDefaultTrigger()
, thegetDefaultTrigger(StreamExecutionEnvironment env)
will be invoked in the default implementation of thegetDefaultTrigger()
.- Specified by:
getDefaultTrigger
in classWindowAssigner<Object,TimeWindow>
-
of
public static SlidingEventTimeWindows of(Duration size, Duration slide)
Creates a newSlidingEventTimeWindows
WindowAssigner
that assigns elements to sliding time windows based on the element timestamp.- Parameters:
size
- The size of the generated windows.slide
- The slide interval of the generated windows.- Returns:
- The time policy.
-
of
public static SlidingEventTimeWindows of(Duration size, Duration slide, Duration offset)
Creates a newSlidingEventTimeWindows
WindowAssigner
that assigns elements to time windows based on the element timestamp and offset.For example, if you want window a stream by hour,but window begins at the 15th minutes of each hour, you can use
of(Duration.ofHours(1), Duration.ofMinutes(15))
, then you will get time windows start at 0:15:00,1:15:00,2:15:00,etc.Rather than that,if you are living in somewhere which is not using UTC±00:00 time, such as China which is using UTC+08:00,and you want a time window with size of one day, and window begins at every 00:00:00 of local time,you may use
of(Duration.ofDays(1), Duration.ofHours(-8))
. The parameter of offset isDuration.ofHours(-8))
since UTC+08:00 is 8 hours earlier than UTC time.- Parameters:
size
- The size of the generated windows.slide
- The slide interval of the generated windows.offset
- The offset which window start would be shifted by.- Returns:
- The time policy.
-
getWindowSerializer
public TypeSerializer<TimeWindow> getWindowSerializer(ExecutionConfig executionConfig)
Description copied from class:WindowAssigner
Returns aTypeSerializer
for serializing windows that are assigned by thisWindowAssigner
.- Specified by:
getWindowSerializer
in classWindowAssigner<Object,TimeWindow>
-
isEventTime
public boolean isEventTime()
Description copied from class:WindowAssigner
Returnstrue
if elements are assigned to windows based on event time,false
otherwise.- Specified by:
isEventTime
in classWindowAssigner<Object,TimeWindow>
-
-