Class TumblingProcessingTimeWindows

  • All Implemented Interfaces:
    Serializable

    public class TumblingProcessingTimeWindows
    extends WindowAssigner<Object,​TimeWindow>
    A WindowAssigner that windows elements into windows based on the current system time of the machine the operation is running on. Windows cannot overlap.

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

    
     DataStream<Tuple2<String, Integer>> in = ...;
     KeyedStream<String, Tuple2<String, Integer>> keyed = in.keyBy(...);
     WindowedStream<Tuple2<String, Integer>, String, TimeWindows> windowed =
       keyed.window(TumblingProcessingTimeWindows.of(Time.of(1, MINUTES), Time.of(10, SECONDS));
     
    See Also:
    Serialized Form
    • Method Detail

      • getSize

        public long getSize()
      • of

        public static TumblingProcessingTimeWindows of​(Time size)
        Creates a new TumblingProcessingTimeWindows 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 TumblingProcessingTimeWindows of​(Time size,
                                                       Time offset)
        Creates a new TumblingProcessingTimeWindows 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.
        Returns:
        The time policy.
      • of

        @PublicEvolving
        public static TumblingProcessingTimeWindows of​(Time size,
                                                       Time offset,
                                                       WindowStagger windowStagger)
        Creates a new TumblingProcessingTimeWindows 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 offset which window start would be shifted by.
        windowStagger - The utility that produces staggering offset in runtime.
        Returns:
        The time policy.
      • 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>