[ 
https://issues.apache.org/jira/browse/FLINK-8384?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16327322#comment-16327322
 ] 

ASF GitHub Bot commented on FLINK-8384:
---------------------------------------

Github user dyanarose commented on a diff in the pull request:

    https://github.com/apache/flink/pull/5295#discussion_r161807828
  
    --- Diff: 
flink-streaming-java/src/main/java/org/apache/flink/streaming/api/windowing/assigners/DynamicEventTimeSessionWindows.java
 ---
    @@ -0,0 +1,101 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one
    + * or more contributor license agreements.  See the NOTICE file
    + * distributed with this work for additional information
    + * regarding copyright ownership.  The ASF licenses this file
    + * to you under the Apache License, Version 2.0 (the
    + * "License"); you may not use this file except in compliance
    + * with the License.  You may obtain a copy of the License at
    + *
    + * http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +
    +package org.apache.flink.streaming.api.windowing.assigners;
    +
    +import org.apache.flink.api.common.ExecutionConfig;
    +import org.apache.flink.api.common.typeutils.TypeSerializer;
    +import 
org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
    +import org.apache.flink.streaming.api.windowing.triggers.Trigger;
    +import 
org.apache.flink.streaming.api.windowing.triggers.TypedEventTimeTrigger;
    +import org.apache.flink.streaming.api.windowing.windows.TimeWindow;
    +
    +import java.util.Collection;
    +import java.util.Collections;
    +
    +/**
    + * A {@link WindowAssigner} that windows elements into sessions based on 
the timestamp of the
    + * elements. Windows cannot overlap.
    + *
    + * <p>For example, in order to window into windows with a dynamic time gap:
    + * <pre> {@code
    + * DataStream<Tuple2<String, Integer>> in = ...;
    + * KeyedStream<String, Tuple2<String, Integer>> keyed = in.keyBy(...);
    + * WindowedStream<Tuple2<String, Integer>, String, TimeWindows> windowed =
    + *   keyed.window(DynamicEventTimeSessionWindows.withDynamicGap({@link 
SessionWindowTimeGapExtractor }));
    + * } </pre>
    + *
    + * @param <T> The type of the input elements
    + */
    +public class DynamicEventTimeSessionWindows<T> extends 
MergingWindowAssigner<T, TimeWindow> {
    +   private static final long serialVersionUID = 1L;
    --- End diff --
    
    The Typed requirement comes from the desire to allow the 
SessionWindowTimeGapExtractor to accept a correctly typed element.
    
    To do that the Assigner itself needs to be typed, which means that the 
trigger needs to be typed and so on.
    
    If the SessionWindowTimeGapExtractor extract method instead took `object`, 
requiring that the implementer cast it, then the new Typed classes wouldn't be 
necessary.
    
    I don't find that to be the most user friendly interface though, when the 
type information is available. But, yeah, I'm not happy with having to 
implement these exact copy classes either...


> 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
(v7.6.3#76005)

Reply via email to