Re: Publishing Sink Task watermarks outside flink

2020-05-18 Thread Timo Walther
Hi Shubham, great that tweaking the JDBC sink helped. Maybe I don't fully understand your logic but: The watermark that you are receiving in an operator should already be the minimum of all subtasks. Because it is sent to all subsequent operators by the precedeing operator. So a watermark ca

Re: Publishing Sink Task watermarks outside flink

2020-05-03 Thread Shubham Kumar
Following up on this, I tried tweaking the Jdbc Sink as Timo suggested and was successful in it. Basically I added a member *long maxWatermarkSeen *in JDBCOutputFormat, so whenever a new record is added to the batch it updates the *maxWatermarkSeen* for this subtask with *org.apache.flink.streamin

Re: Publishing Sink Task watermarks outside flink

2020-04-28 Thread Shubham Kumar
Hi Timo, Yeah, I got the idea of getting access to timers through process function and had the same result which you explained that is a side output doesn't guarantee that the data is written out to sink. (so maybe Fabian in that post pointed out something else which I am missing). If I am correct

Re: Publishing Sink Task watermarks outside flink

2020-04-28 Thread Timo Walther
Hi Shubham, you can call stream.process(...). The context of ProcessFunction gives you access to TimerService which let's you access the current watermark. I'm assuming your are using the Table API? As far as I remember, watermark are travelling through the stream even if there is no time-ba

Publishing Sink Task watermarks outside flink

2020-04-28 Thread Shubham Kumar
Hi everyone, I have a flink application having kafka sources which calculates some stats based on it and pushes it to JDBC. Now, I want to know till what timestamp is the data completely pushed in JDBC (i.e. no more data will be pushed to timestamp smaller or equal than this). There doesn't seem t