Class TumblingEventTimeWindows

  • All Implemented Interfaces:
    Serializable

    @PublicEvolving
    public class TumblingEventTimeWindows
    extends WindowAssigner<Object,​TimeWindow>
    A WindowAssigner that windows elements into windows based on the timestamp of the elements. Windows cannot overlap.

    For example, in order to window into windows of 1 minute:

    
     DataStream<Tuple2<String, Integer>> in = ...;
     KeyedStream<Tuple2<String, Integer>, String> keyed = in.keyBy(...);
     WindowedStream<Tuple2<String, Integer>, String, TimeWindow> windowed =
       keyed.window(TumblingEventTimeWindows.of(Time.minutes(1)));
     
    See Also:
    Serialized Form
    • Constructor Detail

      • TumblingEventTimeWindows

        protected TumblingEventTimeWindows​(long size,
                                           long offset,
                                           WindowStagger windowStagger)
    • Method Detail

      • of

        public static TumblingEventTimeWindows of​(Time size)
        Creates a new TumblingEventTimeWindows WindowAssigner that assigns elements to time windows based on the element timestamp.
        Parameters:
        size - The size of the generated windows.
        Returns:
        The time policy.
      • of

        public static TumblingEventTimeWindows of​(Time size,
                                                  Time offset)
        Creates a new TumblingEventTimeWindows 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(Time.hours(1),Time.minutes(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(Time.days(1),Time.hours(-8)). The parameter of offset is Time.hours(-8)) since UTC+08:00 is 8 hours earlier than UTC time.

        Parameters:
        size - The size of the generated windows.
        offset - The offset which window start would be shifted by.
      • of

        @PublicEvolving
        public static TumblingEventTimeWindows of​(Time size,
                                                  Time offset,
                                                  WindowStagger windowStagger)
        Creates a new TumblingEventTimeWindows WindowAssigner that assigns elements to time windows based on the element timestamp, offset and a staggering offset, depending on the staggering policy.
        Parameters:
        size - The size of the generated windows.
        offset - The globalOffset which window start would be shifted by.
        windowStagger - The utility that produces staggering offset in runtime.
      • getWindowSerializer

        public org.apache.flink.api.common.typeutils.TypeSerializer<TimeWindow> getWindowSerializer​(org.apache.flink.api.common.ExecutionConfig executionConfig)
        Description copied from class: WindowAssigner
        Returns a TypeSerializer for serializing windows that are assigned by this WindowAssigner.
        Specified by:
        getWindowSerializer in class WindowAssigner<Object,​TimeWindow>