GitHub user GJL opened a pull request: https://github.com/apache/flink/pull/4910
[FLINK-7784] [kafka-producer] Don't fail TwoPhaseCommitSinkFunction when failing to commit during job recovery ## What is the purpose of the change This makes it possible to configure the TwoPhaseCommitSinkFunction's behaviour w.r.t. transaction timeouts. ## Brief change log - *Introduce transaction timeouts to TwoPhaseCommitSinkFunction.* - *Timeout can be used to generate warnings if the transaction's age approaches the timeout.* - *If an exception is thrown during job recovery, the sink can be configured not to propagate the exception and instead log it on ERROR level.* ## Verifying this change This change added tests and can be verified as follows: - *Extended unit tests for TwoPhaseCommitSinkFunction to test added functionality* - *Manually verified the change by running a job with a FlinkKafka011Producer with checkpoint interval 27000 and transaction.timeout.ms = 30000. Warnings were generated correctly.* ## Does this pull request potentially affect one of the following parts: - Dependencies (does it add or upgrade a dependency): (yes / **no**) - The public API, i.e., is any changed class annotated with `@Public(Evolving)`: (**yes** / no) - The serializers: (yes / **no** / don't know) - The runtime per-record code paths (performance sensitive): (yes / **no** / don't know) - Anything that affects deployment or recovery: JobManager (and its components), Checkpointing, Yarn/Mesos, ZooKeeper: (yes / **no** / don't know) ## Documentation - Does this pull request introduce a new feature? (**yes** / no) - If yes, how is the feature documented? (not applicable / docs / **JavaDocs** / not documented) You can merge this pull request into a Git repository by running: $ git pull https://github.com/GJL/flink FLINK-7784 Alternatively you can review and apply these changes as the patch at: https://github.com/apache/flink/pull/4910.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #4910 ---- commit 23ae221eb854ce12572988aed5c018aac8919af7 Author: gyao <g...@data-artisans.com> Date: 2017-10-26T17:17:55Z [FLINK-7784] [kafka011-producer] Make TwoPhaseCommitSinkFunction aware of transaction timeouts. TwoPhaseCommitSinkFunction allows to configure a transaction timeout. The timeout can be used to log warnings if the transaction's age is appraoching the timeout, and it can be used to swallow exceptions that are likely irrecoverable. This commit also integrates these changes to the FlinkKafkaProducer011. commit 43103c1fb61a6bc1aec6b19c0253fcca281cfba5 Author: gyao <g...@data-artisans.com> Date: 2017-10-26T17:25:35Z [hotfix] [kafka-tests] Clean up FlinkKafkaProducer011Tests ---- ---