Uploaded image for project: 'Flink'
  1. Flink
  2. FLINK-24738

Fail during announcing buffer size to released local channel

    XMLWordPrintableJSON

Details

    Description

      Since we can release all resources not only when the mailbox would be finished but also from the mailbox:

      org.apache.flink.runtime.io.network.partition.consumer.LocalInputChannel.releaseAllResources(LocalInputChannel.java:331)
      org.apache.flink.runtime.io.network.partition.consumer.SingleInputGate.transformEvent(SingleInputGate.java:808)
      org.apache.flink.runtime.io.network.partition.consumer.SingleInputGate.transformToBufferOrEvent(SingleInputGate.java:757)
      org.apache.flink.runtime.io.network.partition.consumer.SingleInputGate.getNextBufferOrEvent(SingleInputGate.java:687)
      org.apache.flink.runtime.io.network.partition.consumer.SingleInputGate.pollNext(SingleInputGate.java:666)
      org.apache.flink.runtime.taskmanager.InputGateWithMetrics.pollNext(InputGateWithMetrics.java:142)
      org.apache.flink.streaming.runtime.io.checkpointing.CheckpointedInputGate.pollNext(CheckpointedInputGate.java:150)
      org.apache.flink.streaming.runtime.io.AbstractStreamTaskNetworkInput.emitNext(AbstractStreamTaskNetworkInput.java:110)
      org.apache.flink.streaming.runtime.io.StreamOneInputProcessor.processInput(StreamOneInputProcessor.java:65)
      org.apache.flink.streaming.runtime.tasks.StreamTask.processInput(StreamTask.java:503)
      org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop(MailboxProcessor.java:203)
      org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop(StreamTask.java:816)
      org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:768)
      org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring(Task.java:958)
      org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke(Task.java:936)
      org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:750)
      org.apache.flink.runtime.taskmanager.Task.run(Task.java:571)
      

      It is possible that after it the BufferDebloater announce the new buffer size which will fail because the channel is released already:

      Caused by: java.lang.IllegalStateException: Channel released.
      	at org.apache.flink.util.Preconditions.checkState(Preconditions.java:193)
      	at org.apache.flink.runtime.io.network.partition.consumer.LocalInputChannel.announceBufferSize(LocalInputChannel.java:354)
      	at org.apache.flink.runtime.io.network.partition.consumer.SingleInputGate.announceBufferSize(SingleInputGate.java:389)
      	at org.apache.flink.runtime.taskmanager.InputGateWithMetrics.announceBufferSize(InputGateWithMetrics.java:102)
      	at org.apache.flink.streaming.runtime.tasks.bufferdebloat.BufferDebloater.recalculateBufferSize(BufferDebloater.java:101)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.debloat(StreamTask.java:801)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.lambda$null$4(StreamTask.java:791)
      	at org.apache.flink.streaming.runtime.tasks.StreamTaskActionExecutor$1.runThrowing(StreamTaskActionExecutor.java:50)
      	at org.apache.flink.streaming.runtime.tasks.mailbox.Mail.run(Mail.java:90)
      	at org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.processMailsWhenDefaultActionUnavailable(MailboxProcessor.java:338)
      	at org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.processMail(MailboxProcessor.java:324)
      	at org.apache.flink.streaming.runtime.tasks.mailbox.MailboxProcessor.runMailboxLoop(MailboxProcessor.java:201)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.runMailboxLoop(StreamTask.java:816)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:768)
      	at org.apache.flink.runtime.taskmanager.Task.runWithSystemExitMonitoring(Task.java:958)
      	at org.apache.flink.runtime.taskmanager.Task.restoreAndInvoke(Task.java:936)
      	at org.apache.flink.runtime.taskmanager.Task.doRun(Task.java:750)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:571)
      	at java.lang.Thread.run(Thread.java:748)
      

       So I think that we should replace `checkState` with `if` for LocalInputChannel#announceBufferSize since released channel is expected here.

      Attachments

        Activity

          People

            akalashnikov Anton Kalashnikov
            akalashnikov Anton Kalashnikov
            Votes:
            0 Vote for this issue
            Watchers:
            2 Start watching this issue

            Dates

              Created:
              Updated:
              Resolved: