Developing Apache Storm Applications
Also available as:
PDF

Implementing Windowing in Trident

Trident processes a stream in batches of tuples for a defined topology. As with core Storm, Trident supports tumbling and sliding windows. Either type of window can be based on processing time, tuple count, or both.

Windowing API for Trident

The common windowing API takes WindowConfig for any supported windowing configuration. It returns a stream of aggregated results based on the given window configuration.

public Stream window(WindowConfig windowConfig, 
                   Fields inputFields, 
                   Aggregator aggregator, 
                   Fields functionFields)

windowConfig can be any of the following:

  • SlidingCountWindow of(int windowCount, int slidingCount)

  • SlidingDurationWindow of(BaseWindowedBolt.Duration windowDuration,

    BaseWindowedBolt.Duration slidingDuration)

  • TumblingCountWindow of(int windowLength)

  • TumblingDurationWindow of(BaseWindowedBolt.Duration windowLength)

Trident windowing APIs also need to implement WindowsStoreFactory, to store received tuples and aggregated values.

Implementing a Tumbling Window

For a tumbling window implementation, tuples are grouped in a single window based on processing time or count. Any tuple belongs to only one window. Here is the API for a tumbling window:

    /**
     * Returns a stream of tuples which are aggregated results of a tumbling window with
            every {@code windowCount} of tuples.
     */
public Stream tumblingWindow(int windowCount,
                          WindowsStoreFactory windowStoreFactory,
                          Fields inputFields, 
                          Aggregator aggregator, 
                          Fields functionFields)
    /**
     * Returns a stream of tuples which are aggregated results of a window that tumbles at
            duration of {@code windowDuration}
     */

public Stream tumblingWindow(BaseWindowedBolt.Duration windowDuration, 
                          WindowsStoreFactory windowStoreFactory, 
                          Fields inputFields, 
                          Aggregator aggregator, 
                          Fields functionFields)

Implementing a Sliding Window

For a sliding window implementation, tuples are grouped in windows that slide for every sliding interval. A tuple can belong to more than one window. Here is the API for a sliding window:

    /**
     * Returns a stream of tuples which are aggregated results of a sliding window with
            every {@code windowCount} of tuples and slides the window after {@code slideCount}.
     */
public Stream slidingWindow(int windowCount, 
                          int slideCount, 
                          WindowsStoreFactory windowStoreFactory,
                          Fields inputFields,
                          Aggregator aggregator, 
                          Fields functionFields)
/**
* Returns a stream of tuples which are aggregated results of a window which slides at
            duration of {@code slidingInterval}
* and completes a window at {@code windowDuration}
*/
                          public Stream slidingWindow( BaseWindowedBolt.Duration windowDuration, 
                          BaseWindowedBolt.Duration slidingInterval,
                          WindowsStoreFactory windowStoreFactory, 
                          Fields inputFields,
                          Aggregator aggregator,
                          Fields functionFields)