[
https://issues.apache.org/jira/browse/FLINK-39167?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
]
Roman Khachatryan updated FLINK-39167:
--------------------------------------
Description:
After FLINK-38939, if the source operator was stopped while waiting for the
first checkpoint then the output needs to be initialized so final watermark can
be emitted; otherwise, final checkpoint might fail with
{code:java}
java.lang.IllegalStateException: null
at org.apache.flink.util.Preconditions.checkState
at
org.apache.flink.streaming.api.operators.SourceOperator.emitLatestWatermark
at
org.apache.flink.streaming.api.operators.SourceOperator.emitNextNotReading
at org.apache.flink.streaming.api.operators.SourceOperator.emitNext
at org.apache.flink.streaming.runtime.io.StreamTaskSourceInput.emitNext
at
org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput
at org.apache.flink.streaming.runtime.tasks.StreamTask.processInput
at
org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop
at org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop
at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke
at org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring
at org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke
at org.apache.flink.runtime.taskmanager.Task.doRun
at org.apache.flink.runtime.taskmanager.Task.run
{code}
was:
After FLINK-38939, if the source operator was stopped while waiting for the
first checkpoint then the output needs to be initialized so final watermark can
be emitted; otherwise, final checkpoint willÂ
{code}
java.lang.IllegalStateException: null
at org.apache.flink.util.Preconditions.checkState
at
org.apache.flink.streaming.api.operators.SourceOperator.emitLatestWatermark
at
org.apache.flink.streaming.api.operators.SourceOperator.emitNextNotReading
at org.apache.flink.streaming.api.operators.SourceOperator.emitNext
at org.apache.flink.streaming.runtime.io.StreamTaskSourceInput.emitNext
at
org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput
at org.apache.flink.streaming.runtime.tasks.StreamTask.processInput
at
org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop
at org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop
at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke
at org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring
at org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke
at org.apache.flink.runtime.taskmanager.Task.doRun
at org.apache.flink.runtime.taskmanager.Task.run
{code}
> Initialize source output before emitting final watermark
> --------------------------------------------------------
>
> Key: FLINK-39167
> URL: https://issues.apache.org/jira/browse/FLINK-39167
> Project: Flink
> Issue Type: Bug
> Components: Runtime / Task
> Affects Versions: 2.3.0
> Reporter: Roman Khachatryan
> Assignee: Roman Khachatryan
> Priority: Major
> Fix For: 2.3.0
>
>
> After FLINK-38939, if the source operator was stopped while waiting for the
> first checkpoint then the output needs to be initialized so final watermark
> can be emitted; otherwise, final checkpoint might fail with
> {code:java}
> java.lang.IllegalStateException: null
> at org.apache.flink.util.Preconditions.checkState
> at
> org.apache.flink.streaming.api.operators.SourceOperator.emitLatestWatermark
> at
> org.apache.flink.streaming.api.operators.SourceOperator.emitNextNotReading
> at org.apache.flink.streaming.api.operators.SourceOperator.emitNext
> at org.apache.flink.streaming.runtime.io.StreamTaskSourceInput.emitNext
> at
> org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput
> at org.apache.flink.streaming.runtime.tasks.StreamTask.processInput
> at
> org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop
> at org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop
> at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke
> at org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring
> at org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke
> at org.apache.flink.runtime.taskmanager.Task.doRun
> at org.apache.flink.runtime.taskmanager.Task.run
> {code}
--
This message was sent by Atlassian Jira
(v8.20.10#820010)