Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-12-01 Thread Hang Ruan
Sorry, I spell it wrong, which I mean the PR. Here it is https://github.com/apache/flink/pull/17276 . Marco Villalobos 于2021年12月1日周三 下午9:18写道: > Thank you. One last question. What is an RP? Where can I read it? > > Marco > > On Nov 30, 2021, at 11:06 PM, Hang Ruan wrote: > > Hi, > > In 1.12.0-

Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-12-01 Thread Marco Villalobos
Thank you. One last question. What is an RP? Where can I read it? Marco > On Nov 30, 2021, at 11:06 PM, Hang Ruan wrote: > > Hi, > > In 1.12.0-1.12.5 versions, committing offset to kafka when the checkpoint is > open is the default behavior in KafkaSourceBuilder. And it can not be changed >

Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-11-30 Thread Hang Ruan
Hi, In 1.12.0-1.12.5 versions, committing offset to kafka when the checkpoint is open is the default behavior in KafkaSourceBuilder. And it can not be changed in KafkaSourceBuilder. By this FLINK-24277 , we could change the behavior. This problem

Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-11-30 Thread Marco Villalobos
Thanks! However, I noticed that KafkaSourceOptions.COMMIT_OFFSETS_ON_CHECKPOINT does not exist in Flink 1.12. Is that property supported with the string "commit.offsets.on.checkpoints"? How do I configure that behavior so that offsets get committed on checkpoints in Flink 1.12 when using the Kaf

Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-11-29 Thread Hang Ruan
Hi, Maybe you can write like this : builder.setProperty(KafkaSourceOptions.COMMIT_OFFSETS_ON_CHECKPOINT.key(), "true"); Other additional properties could be found here : https://nightlies.apache.org/flink/flink-docs-release-1.14/docs/connectors/datastream/kafka/#additional-properties Marco Villa

Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-11-29 Thread Marco Villalobos
Thank you for the information. That still does not answer my question though. How do I configure Flink in 1.12 using the KafkaSourceBuilder so that consumer should commit offsets back to Kafka on checkpoints? FlinkKafkaConsumer#setCommitOffsetsOnCheckpoints(boolean) has this method. But now tha

Re: How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-11-29 Thread Caizhi Weng
Hi! Flink 1.14 release note states about this. See [1]. [1] https://nightlies.apache.org/flink/flink-docs-release-1.14/zh/release-notes/flink-1.14/#deprecate-flinkkafkaconsumer Marco Villalobos 于2021年11月30日周二 上午7:12写道: > Hi everybody, > > I am using Flink 1.12 and migrating my code from using

How do you configure setCommitOffsetsOnCheckpoints in Flink 1.12 when using KafkaSourceBuilder?

2021-11-29 Thread Marco Villalobos
Hi everybody, I am using Flink 1.12 and migrating my code from using FlinkKafkaConsumer to using the KafkaSourceBuilder. FlinkKafkaConsumer has the method /** > * Specifies whether or not the consumer should commit offsets back to > Kafka on checkpoints. > * This setting will only have effect