[ https://issues.apache.org/jira/browse/FLINK-7784?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16225068#comment-16225068 ]
ASF GitHub Bot commented on FLINK-7784: --------------------------------------- Github user aljoscha commented on a diff in the pull request: https://github.com/apache/flink/pull/4910#discussion_r147727170 --- Diff: flink-streaming-java/src/main/java/org/apache/flink/streaming/api/functions/sink/TwoPhaseCommitSinkFunction.java --- @@ -58,18 +61,37 @@ extends RichSinkFunction<IN> implements CheckpointedFunction, CheckpointListener { - private static final Logger LOG = LoggerFactory.getLogger(TwoPhaseCommitSinkFunction.class); + private final Logger log; - protected final ListStateDescriptor<State<TXN, CONTEXT>> stateDescriptor; + private final Clock clock; - protected final LinkedHashMap<Long, TXN> pendingCommitTransactions = new LinkedHashMap<>(); + protected final LinkedHashMap<Long, TransactionHolder<TXN>> pendingCommitTransactions = new LinkedHashMap<>(); - @Nullable - protected TXN currentTransaction; protected Optional<CONTEXT> userContext; protected ListState<State<TXN, CONTEXT>> state; + private final ListStateDescriptor<State<TXN, CONTEXT>> stateDescriptor; + + private TransactionHolder<TXN> currentTransaction; + + /** + * Specifies the maximum time a transaction should remain open. + */ + private long transactionTimeout = Long.MAX_VALUE; + + /** + * If true, any exception thrown in {@link #recoverAndCommit(Object)} will be caught instead of + * propagated. + */ + private boolean failureOnCommitAfterTransactionTimeoutDisabled; + + /** + * If a transaction's elapsed time reaches this percentage of the transactionTimeout, a warning + * message will be logged. Value must be in range [0,1]. Negative value disables warnings. + */ + private double transactionTimeoutWarningRatio = -1; --- End diff -- I tend to agree with @pnowojski about the API surface but I think in this case this is a valid safety net for possible future transaction sinks. > Don't fail TwoPhaseCommitSinkFunction when failing to commit > ------------------------------------------------------------ > > Key: FLINK-7784 > URL: https://issues.apache.org/jira/browse/FLINK-7784 > Project: Flink > Issue Type: Bug > Components: DataStream API > Affects Versions: 1.4.0 > Reporter: Aljoscha Krettek > Assignee: Gary Yao > Priority: Blocker > Fix For: 1.4.0 > > > Currently, {{TwoPhaseCommitSinkFunction}} will fail if committing fails > (either when doing it via the completed checkpoint notification or when > trying to commit after restoring after failure). This means that the job will > go into an infinite recovery loop because we will always keep failing. > In some cases it might be better to ignore those failures and keep on > processing and this should be the default. We can provide an option that > allows failing the sink on failing commits. -- This message was sent by Atlassian JIRA (v6.4.14#64029)