Spike in checkpoint start delay every 15 minutes

2022-06-14 Thread Jai Patel
We've noticed a spike in the start delays in our incremental checkpoints every 15 minutes. The Flink job seems to start out smooth, with checkpoints in in the 15s range and negligible start delays. Then every 3rd or 4th checkpoint has a long start delay (~2-3 minutes). Teh checkpoints in

Re: UnsupportedOperationException: The end timestamp of a processing-time window cannot become earlier than the current processing time by merging.

2022-04-15 Thread Jai Patel
15, 2022 at 5:52 PM Jai Patel wrote: > Here's our custom trigger. We thought about switching to > ProcessingTimeoutTrigger.of(CountTrigger.of(100, Time.ofMinutes(1)). But > I'm not sure that'll trigger properly when the window closes. > > Thanks

Re: UnsupportedOperationException: The end timestamp of a processing-time window cannot become earlier than the current processing time by merging.

2022-04-15 Thread Jai Patel
{ processingTimeTrigger.clear(window, ctx); countTrigger.clear(window, ctx); } } On Fri, Apr 15, 2022 at 2:57 PM Jai Patel wrote: > We are encountering the following error when running our Flink job. We > have several processing windows, but it appears to be r

UnsupportedOperationException: The end timestamp of a processing-time window cannot become earlier than the current processing time by merging.

2022-04-15 Thread Jai Patel
We are encountering the following error when running our Flink job. We have several processing windows, but it appears to be related to a TumblingProcessingTimeWindow. Checkpoints are failing to complete midway. The code block for the window is: .keyBy(order -> getKey(order))

Re: java.lang.Exception: Job leader for job id 0efd8681eda64b072b72baef58722bc0 lost leadership.

2022-04-15 Thread Jai Patel
Hi Nico, Wanted to close the loop here. We did end up find a number of problems in our code: 1. Our operator was slow. It was iterating over several large Protobufs in a MapState then filtering it down to 1. We were able to identify that one up-front and significantly improve the runtime of

Re: java.lang.Exception: Job leader for job id 0efd8681eda64b072b72baef58722bc0 lost leadership.

2022-02-23 Thread Jai Patel
of the Flink State or is the data buffered in between? Thanks. Jai On Wed, Feb 23, 2022 at 12:55 AM Nicolaus Weidner < nicolaus.weid...@ververica.com> wrote: > Hi Jai, > > On Tue, Feb 22, 2022 at 9:19 PM Jai Patel > wrote: > >> It seems like the errors are similar to those

java.lang.Exception: Job leader for job id 0efd8681eda64b072b72baef58722bc0 lost leadership.

2022-02-22 Thread Jai Patel
We are developing a new feature for our Flink application that relies upon joining multiple Kafka Streams and uses Flink State to handle joining information asynchronously. Recently as the volume of data has been growing, we've been noticing a couple exceptions while trying to enable the feature.

Change In Behavior for EventTimeSessionWindow between Flink 1.11.1 and 1.14.0

2021-11-11 Thread Jai Patel
Hey all, I observed what appears to be a change in behavior for EventTimeSessionWindows when upgrading from 1.11.1 to 1.14.0. This was identified in a unit test. For a window with a defined time gap of 10 seconds. - Publish KEY_1 with eventtime 1 second - Publish KEY_1 with eventtime 3