flink-issues mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From "Tzu-Li (Gordon) Tai (JIRA)" <j...@apache.org>
Subject [jira] [Comment Edited] (FLINK-3398) Flink Kafka consumer should support auto-commit opt-outs
Date Thu, 02 Feb 2017 10:13:51 GMT

    [ https://issues.apache.org/jira/browse/FLINK-3398?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15849733#comment-15849733
] 

Tzu-Li (Gordon) Tai edited comment on FLINK-3398 at 2/2/17 10:13 AM:
---------------------------------------------------------------------

I think we should keep this JIRA instead of merging it into FLINK-4280.
They try to address different new features: FLINK-4280 is for more flexible start position
configuration independent from Kafka offsets.
This JIRA is for opt-out from offset committing on Flink's checkpoints.

I'll move the respective proposed API for commit opt-out here:

{code}
Properties props = new Properties();
...
FlinkKafkaConsumer09 kafka = new FlinkKafkaConsumer09("topic", schema, props);
kafka.setEnableOffsetCommittingOnCheckpoints(boolean); // if true (checkpointing should be
enabled), overrides periodic checkpointing if "enable.auto.commit" is set in props.
{code}

So the scenarios breaks down into:

1. {{props.put("auto.commit.enable", "true")}} & {{setEnableOffsetCommittingOnCheckpoints(false)}}:
Perform auto committing with the internal client (or in 0.8 case our own periodic committer).

2. {{props.put("auto.commit.enable", "true")}} & {{setEnableOffsetCommittingOnCheckpoints(true)}}:
Overrides whatever value in props for "auto.commit.enable", only commit on checkpoints. If
checkpointing isn't enabled, the consumer fails due to conflicting configuration.

3. {{props.put("auto.commit.enable", "false")}} & {{setEnableOffsetCommittingOnCheckpoints(false)}}:
No offset committing at all.



was (Author: tzulitai):
I think we should keep this JIRA instead of merging it into FLINK-4280.
They try to address different new features: FLINK-4280 is for more flexible start position
configuration independent from Kafka offsets.
This JIRA is for opt-out from offset committing on Flink's checkpoints.

I'll move the respective proposed API for commit opt-out here:

{code}
Properties props = new Properties();
...
FlinkKafkaConsumer09 kafka = new FlinkKafkaConsumer09("topic", schema, props);
kafka.setEnableOffsetCommittingOnCheckpoints(boolean); // if true (checkpointing should be
enabled), overrides periodic checkpointing if "enable.auto.commit" is set in props.
{code}

So the scenarios breaks down into:

1. {{props.put("auto.commit.enable", "true")}} & {{setEnableOffsetCommittingOnCheckpoints(false)}}:
Perform auto committing with the internal client (or in 0.8 case our own periodic committer).

2. {{props.put("auto.commit.enable", "true")}} & {{setEnableOffsetCommittingOnCheckpoints(true)}}:
Overrides whatever value in props for "auto.commit.enable", only commit on checkpoints. If
checkpointing isn't enabled, the consumer fails due to conflicting configuration.

3. {{props.put("auto.commit.enable", "false")}} & {{setEnableOffsetCommittingOnCheckpoints(false)}}:
No offset committing at all.


> Flink Kafka consumer should support auto-commit opt-outs
> --------------------------------------------------------
>
>                 Key: FLINK-3398
>                 URL: https://issues.apache.org/jira/browse/FLINK-3398
>             Project: Flink
>          Issue Type: Improvement
>          Components: Kafka Connector
>            Reporter: Shikhar Bhushan
>            Assignee: Tzu-Li (Gordon) Tai
>            Priority: Critical
>             Fix For: 1.3.0
>
>
> Currently the Kafka source will commit consumer offsets to Zookeeper, either upon a checkpoint
if checkpointing is enabled, otherwise periodically based on {{auto.commit.interval.ms}}
> It should be possible to opt-out of committing consumer offsets to Zookeeper. Kafka has
this config as {{auto.commit.enable}} (0.8) and {{enable.auto.commit}} (0.9).



--
This message was sent by Atlassian JIRA
(v6.3.15#6346)

Mime
View raw message