You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@flink.apache.org by "ASF GitHub Bot (JIRA)" <ji...@apache.org> on 2015/09/23 21:27:04 UTC

[jira] [Commented] (FLINK-2753) Add new window API to streaming API

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

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

GitHub user StephanEwen opened a pull request:

    https://github.com/apache/flink/pull/1175

    [FLINK-2753] [streaming] [api breaking] Add first parts of new window API for key grouped windows

    This follows the API design outlined in https://cwiki.apache.org/confluence/display/FLINK/Streams+and+Operations+on+Streams
    
    This change is API breaking because it adds new generic type parameters to Java and Scala classes, breaking binary compatibility.
    
    This new API uses in the background the dedicated operators added in a previous pull request, which improved robustness and performance and fixed several correctness bugs.
    
    ### New window API
    
    ```java
    DataStream<MyType> stream = ...;
    
    stream.keyBy("id")
          .window(Time.of(5, SECONDS))
          .reduceWindow( (a, b) -> a.fuse(b) )
    ```
    
    The Pull Request also introduces the first parts of defining the programs Time characteristic on the StreamExecutionEnvironment:
      - *Processing time*
      - *Ingestion time*
      - *Event time*
    
    Eventually, the other window mechanisms should be replaced by this:
    
      - Windows per key are covered in this new mechanism
      - Global windows (global parallel discretization) were decided to be dropped for safety and replaced by single-group windows (non parallel) with possible parallel pre-aggregation (for aligned time windows)
    
    ### Follow-up
    
      - Adding the generic window operator based on generic policies (adaption of the current windowing system by @aljoscha ), see https://issues.apache.org/jira/browse/FLINK-2677
      - Automatically activating and configuring Watermark generation at the data sources
      - Adding a dedicated Event Time Window operator
      - Adding the single-group non-parallel windows

You can merge this pull request into a Git repository by running:

    $ git pull https://github.com/StephanEwen/incubator-flink windows

Alternatively you can review and apply these changes as the patch at:

    https://github.com/apache/flink/pull/1175.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

    This closes #1175
    
----
commit 61c3666a272413940e65a2195b87c7472a8e8806
Author: Stephan Ewen <se...@apache.org>
Date:   2015-09-23T10:05:54Z

    [FLINK-2753] [streaming] [api breaking] Add first parts of new window API for key grouped windows
    
    This follows the API design outlined in https://cwiki.apache.org/confluence/display/FLINK/Streams+and+Operations+on+Streams
    
    This is API breaking because it adds new generic type parameters to Java and Scala classes, breaking binary compatibility.

----


> Add new window API to streaming API
> -----------------------------------
>
>                 Key: FLINK-2753
>                 URL: https://issues.apache.org/jira/browse/FLINK-2753
>             Project: Flink
>          Issue Type: Sub-task
>          Components: Streaming
>    Affects Versions: 0.10
>            Reporter: Stephan Ewen
>            Assignee: Stephan Ewen
>             Fix For: 0.10
>
>
> The API integration should follow the API design as documented here:
> https://cwiki.apache.org/confluence/display/FLINK/Streams+and+Operations+on+Streams
> This issue needs:
>   - Add new {{KeyedWindowDataStream}}, created by calling{{window()}} on the {{KeyedDataStream}}.
>   - Add a utility that converts Window Policies into concrete window implementations. This is important to realize special case implementations that do not directly use generic mechanisms implemented by window policies.
>   - Instantiating the operators (dedicated and generic) from the window policies
>   - I will add a stub for the new Window Policy classes, based on the existing policy classes.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)