[
https://issues.apache.org/jira/browse/FLINK-8384?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16323724#comment-16323724
]
Dyana Rose commented on FLINK-8384:
-----------------------------------
For preliminary review and comment while I get the documentation together:
https://github.com/apache/flink/compare/master...SaleCycle:dynamic-session-window-gaps
*What's new:*
* SessionWindowTimeGapExtractor<T> - Generic Interface with one extract method
that returns the time gap
* DynamicEventTimeSessionWindows<T> - Generic event time session window
* DynamicProcessingTimeSessionWindows<T> - Generic processing time session
window
* TypedEventTimeTrigger<T> - Generic event time trigger
* TypedProcessingTimeTrigger<T> - Generic processing time trigger
* Tests for all the above
In order to be able to type the elements passed to the
SessionWindowTimeGapExtractor, I needed to implement a typed versions of both
the EventTimeTrigger and the ProcessingTimeTrigger (which I've simply named
TypedEventTimeTrigger and TypedProcessingTimeTrigger). Other than the generic
added to the class declaration, these are both exactly the same as their
non-generic counterparts (and so are their corresponding test files). I'm not
terrifically happy about that, but felt it was the least invasive
implementation for the existing code.
*Behaviour of dynamic gaps within existing sessions:*
* scenario 1 - the new timeout is prior to the old timeout. The old timeout
(the furthest in the future) is respected.
* scenario 2 - the new timeout is after the old timeout. The new timeout is
respected.
* scenario 3 - a session is in flight, a new timeout is calculated, however no
new events arrive for that session after the new timeout is calculated. This
session will not have its timeout changed
> Session Window Assigner with Dynamic Gaps
> -----------------------------------------
>
> Key: FLINK-8384
> URL: https://issues.apache.org/jira/browse/FLINK-8384
> Project: Flink
> Issue Type: Improvement
> Components: Streaming
> Reporter: Dyana Rose
> Priority: Minor
>
> *Reason for Improvement*
> Currently both Session Window assigners only allow a static inactivity gap.
> Given the following scenario, this is too restrictive:
> * Given a stream of IoT events from many device types
> * Assume each device type could have a different inactivity gap
> * Assume each device type gap could change while sessions are in flight
> By allowing dynamic inactivity gaps, the correct gap can be determined in the
> [assignWindows
> function|https://github.com/apache/flink/blob/master/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/windowing/assigners/EventTimeSessionWindows.java#L59]
> by passing the element currently under consideration, the timestamp, and the
> context to a user defined function. This eliminates the need to create
> unwieldy work arounds if you only have static gaps.
> Dynamic Session Window gaps should be available for both Event Time and
> Processing Time streams.
> (short preliminary discussion:
> https://lists.apache.org/thread.html/08a011c0039e826343e9be0b1bb4e0000cfc2e12976bc65f8a43ee88@%3Cdev.flink.apache.org%3E)
--
This message was sent by Atlassian JIRA
(v6.4.14#64029)