Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-04-05 Thread mejri houssem
Hello, In this paraphrase of the documentation [1], it is mentioned that <<*Kafka source does not **rely on committed offsets for fault tolerance. Committing offset is only for exposing the progress of consumer and consuming group for monitoring*>>. Can someone explain please why the kafka sour

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-27 Thread Andrew Otto
I learned about this a couple of years ago when I was investigating the feasibility of a "Kafka stretch" cluster. From that email thread : > I got excited about this possibility, only to learn that Flink's KafkaSource does not use

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-26 Thread Gabor Somogyi
> Are they two different things? There are no consumer and broker offsets, there are offsets which belong to a topic + partition pair. > And which offset is saved in the checkpoint/savepoint? Which Flink thinks is processed already. Regarding the PROD deploy now you know the risks so feel free to

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-26 Thread mejri houssem
Hello Gabor, Thanks for your response. I just want to clarify one thing: is there any difference between the Kafka source offset and the Kafka broker offset? Are they two different things? And which offset is saved in the checkpoint/savepoint? For our use case, we intend to take a savepoint only

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-25 Thread Gabor Somogyi
In short it's encouraged to use savepoint because of the following situation: * You start processing data from offset 0 * 2 savepoints created, one with offset 10, another with 20 * This timeframe Kafka has offset 20 since that's the last processed * At offset 30 you realize that data processed bet

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-25 Thread mejri houssem
Hello, Is there any further clarification or explanation regarding the subject, please? Best regards. Le mer. 19 mars 2025 à 15:02, mejri houssem a écrit : > Hello, > > So if I understand you well, I cannot rely on the kafka broker offset to > achieve at-least-once guarantee. Without checkpoi

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-19 Thread mejri houssem
Hello, So if I understand you well, I cannot rely on the kafka broker offset to achieve at-least-once guarantee. Without checkpoint/savepoint enabled, that would not be possible. Best regards Le mer. 19 mars 2025 à 12:00, Ahmed Hamdy a écrit : > Hi Mejri, > Not exactly, you can still rely on

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-19 Thread Ahmed Hamdy
Hi Mejri, Not exactly, you can still rely on savepoint to restart/redeploy the job from the latest offset recorded in Flink, my reply was regarding your question if you can replace that and just depend on the committed offsets in the kafka broker. For at-least-once semantic savepoints and checkpoin

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-18 Thread mejri houssem
Hello Ahmed, Thanks for the response. Does that mean checkpoints and savepoints have nothing to do with the at-least-once guarantee, since it depends solely on the starting offset configuration? Best Regards Le mar. 18 mars 2025 à 23:59, Ahmed Hamdy a écrit : > Hi Mejri > > > I’m wondering if

Re: Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-18 Thread Ahmed Hamdy
Hi Mejri > I’m wondering if this is strictly necessary, since the Kafka broker itself keeps track of offsets (i am not mistaken). In other words, if we redeploy the job, will it automatically resume from the last Kafka offset, or should we still rely on Flink’s checkpoint/savepoint mechanism to en

Clarification on checkpoint/savepoint usage with Kafka (and RabbitMQ)

2025-03-18 Thread mejri houssem
Hello everyone, We have a stateless Flink job that uses a Kafka source with at-least-once guarantees. We’ve enabled checkpoints so that, in the event of a restart, Flink can restore from the last committed offset stored in a successful checkpoint. Now we’re considering enabling savepoints for our

Checkpoint/ Savepoint usage

2018-06-13 Thread Rinat
Hi mates, on my way of using BucketingSink, I've decided to enable checkpointing, to prevent hanging of files in open state on job failure. But it seems, that I’m not properly understood the meaning of checkpointing … I’ve enabled the fs backend for checkpoints, and while job is working everythi