Re: Synchronization across tasks using checkpoint barriers

2022-02-17 Thread Gopi Krishna M
Just found that we can use prepareSnapshotPreBarrier in AbstractStreamOperator to achieve this. On Tue, Feb 15, 2022 at 9:13 AM

Re: Synchronization across tasks using checkpoint barriers

2022-02-14 Thread Gopi Krishna M
CheckpointedFunction docs mention the following - > The snapshotState(FunctionSnapshotContext) >

Re: Synchronization across tasks using checkpoint barriers

2022-02-14 Thread Gopi Krishna M
On Mon, Feb 14, 2022 at 10:00 PM Niklas Semmler wrote: > So, you want to send basically the last message before the barrier? > Yes. > > Can you not instead send the first message after the barrier? From a first > glance this sounds easier. > I'm not sure if this will help me synchronize the sin

Re: Synchronization across tasks using checkpoint barriers

2022-02-14 Thread Niklas Semmler
So, you want to send basically the last message before the barrier? Can you not instead send the first message after the barrier? From a first glance this sounds easier. Can you share what you are trying to accomplish? Best regards, Niklas > On 14. Feb 2022, at 17:04, Gopi Krishna M wrote: >

Re: Synchronization across tasks using checkpoint barriers

2022-02-14 Thread Gopi Krishna M
Thanks Niklas! This helps with synchronizing uploads across partitioned tasks. The next step is to pass the handle to this upload to the sink which should be part of the same checkpoint. Is it possible to do the following: 1. Keep reducing the events to keyedStore. 2. On snapshotState: upload the

Re: Synchronization across tasks using checkpoint barriers

2022-02-14 Thread Niklas Semmler
Hi Gopi, You can implement CheckpointedFunction and use the method snapshotState(FunctionSnapshotContext) to upload state on each checkpoint. https://nightlies.apache.org/flink/flink-docs-release-1.14/api/java/org/apache/flink/streaming/api/checkpoint/CheckpointedFunction.html Make sure, you d

Synchronization across tasks using checkpoint barriers

2022-02-13 Thread Gopi Krishna M
Hi, In my flink operators, I need to connect to an external service to update state. I was thinking that the updates to the external service can be synchronized via checkpoint barriers. The topology of the stream is a source, then a single stage of operator replicas handling different partitions,