[ 
https://issues.apache.org/jira/browse/BEAM-2930?focusedWorklogId=135089&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-135089
 ]

ASF GitHub Bot logged work on BEAM-2930:
----------------------------------------

                Author: ASF GitHub Bot
            Created on: 15/Aug/18 17:46
            Start Date: 15/Aug/18 17:46
    Worklog Time Spent: 10m 
      Work Description: lukecwik commented on issue #6208: [WIP] [BEAM-2930] 
Side input support for Flink portable streaming.
URL: https://github.com/apache/beam/pull/6208#issuecomment-413277807
 
 
   @tweise You need to use the WindowMappingFn to figure out what side input is 
being asked for. If the side input isn't ready you should store the elements 
into state and set a timer. The simplest pushback solution would be to set a 
processing time timer that would then read those elements and check if they are 
ready, if so process them otherwise keep them in state. You could do better 
with map state or timers with payloads and you could also do better if you 
could use a watermark based timer based upon the side input PCollection.
   
   The windowing strategy on the side input PCollection tells you the [allowed 
lateness](https://github.com/apache/beam/blob/e2583f5e73de50f8af128ecaa331a2e1046d2b08/model/pipeline/src/main/proto/beam_runner_api.proto#L688).
 Once the watermark + allowed lateness of the side input PCollection surpasses 
it, the window becomes ready.
   
   This is how the existing implementation worked using 
[PushbackSideInputDoFnRunner](https://github.com/apache/beam/blob/b87aa6037b37aab95702fd1ace40b835f0f66f55/runners/core-java/src/main/java/org/apache/beam/runners/core/PushbackSideInputDoFnRunner.java)
 and the 
[ReadyCheckingSideInputReader](https://github.com/apache/beam/blob/b87aa6037b37aab95702fd1ace40b835f0f66f55/runners/core-java/src/main/java/org/apache/beam/runners/core/ReadyCheckingSideInputReader.java).

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


Issue Time Tracking
-------------------

    Worklog Id:     (was: 135089)
    Time Spent: 3h  (was: 2h 50m)

> Flink support for portable side input
> -------------------------------------
>
>                 Key: BEAM-2930
>                 URL: https://issues.apache.org/jira/browse/BEAM-2930
>             Project: Beam
>          Issue Type: Sub-task
>          Components: runner-flink
>            Reporter: Henning Rohde
>            Priority: Major
>              Labels: portability
>          Time Spent: 3h
>  Remaining Estimate: 0h
>




--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to