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

HiveTableSourceITCase.testStreamPartitionReadByCreateTime times out

    XMLWordPrintableJSON

Details

    Description

      https://dev.azure.com/apache-flink/apache-flink/_build/results?buildId=9808&view=logs&j=fc5181b0-e452-5c8f-68de-1097947f6483&t=62110053-334f-5295-a0ab-80dd7e2babbf

      2020-11-19T10:34:23.5591765Z [ERROR] Tests run: 18, Failures: 0, Errors: 1, Skipped: 0, Time elapsed: 192.243 s <<< FAILURE! - in org.apache.flink.connectors.hive.HiveTableSourceITCase
      2020-11-19T10:34:23.5593193Z [ERROR] testStreamPartitionReadByCreateTime(org.apache.flink.connectors.hive.HiveTableSourceITCase)  Time elapsed: 120.075 s  <<< ERROR!
      2020-11-19T10:34:23.5593929Z org.junit.runners.model.TestTimedOutException: test timed out after 120000 milliseconds
      2020-11-19T10:34:23.5594321Z 	at java.lang.Thread.sleep(Native Method)
      2020-11-19T10:34:23.5594777Z 	at org.apache.flink.streaming.api.operators.collect.CollectResultFetcher.sleepBeforeRetry(CollectResultFetcher.java:231)
      2020-11-19T10:34:23.5595378Z 	at org.apache.flink.streaming.api.operators.collect.CollectResultFetcher.next(CollectResultFetcher.java:119)
      2020-11-19T10:34:23.5596001Z 	at org.apache.flink.streaming.api.operators.collect.CollectResultIterator.nextResultFromFetcher(CollectResultIterator.java:103)
      2020-11-19T10:34:23.5596610Z 	at org.apache.flink.streaming.api.operators.collect.CollectResultIterator.hasNext(CollectResultIterator.java:77)
      2020-11-19T10:34:23.5597218Z 	at org.apache.flink.table.planner.sinks.SelectTableSinkBase$RowIteratorWrapper.hasNext(SelectTableSinkBase.java:115)
      2020-11-19T10:34:23.5597811Z 	at org.apache.flink.table.api.internal.TableResultImpl$CloseableRowIteratorWrapper.hasNext(TableResultImpl.java:355)
      2020-11-19T10:34:23.5598555Z 	at org.apache.flink.connectors.hive.HiveTableSourceITCase.fetchRows(HiveTableSourceITCase.java:653)
      2020-11-19T10:34:23.5599407Z 	at org.apache.flink.connectors.hive.HiveTableSourceITCase.testStreamPartitionReadByCreateTime(HiveTableSourceITCase.java:594)
      2020-11-19T10:34:23.5599982Z 	at sun.reflect.NativeMethodAccessorImpl.invoke0(Native Method)
      2020-11-19T10:34:23.5600393Z 	at sun.reflect.NativeMethodAccessorImpl.invoke(NativeMethodAccessorImpl.java:62)
      2020-11-19T10:34:23.5600865Z 	at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43)
      2020-11-19T10:34:23.5601300Z 	at java.lang.reflect.Method.invoke(Method.java:498)
      2020-11-19T10:34:23.5601713Z 	at org.junit.runners.model.FrameworkMethod$1.runReflectiveCall(FrameworkMethod.java:50)
      2020-11-19T10:34:23.5602211Z 	at org.junit.internal.runners.model.ReflectiveCallable.run(ReflectiveCallable.java:12)
      2020-11-19T10:34:23.5602688Z 	at org.junit.runners.model.FrameworkMethod.invokeExplosively(FrameworkMethod.java:47)
      2020-11-19T10:34:23.5603181Z 	at org.junit.internal.runners.statements.InvokeMethod.evaluate(InvokeMethod.java:17)
      2020-11-19T10:34:23.5603753Z 	at org.junit.internal.runners.statements.FailOnTimeout$CallableStatement.call(FailOnTimeout.java:298)
      2020-11-19T10:34:23.5604308Z 	at org.junit.internal.runners.statements.FailOnTimeout$CallableStatement.call(FailOnTimeout.java:292)
      2020-11-19T10:34:23.5604780Z 	at java.util.concurrent.FutureTask.run(FutureTask.java:266)
      2020-11-19T10:34:23.5605114Z 	at java.lang.Thread.run(Thread.java:748)
      2020-11-19T10:34:23.5605299Z 
      2020-11-19T10:34:24.4180149Z [INFO] Running org.apache.flink.connectors.hive.TableEnvHiveConnectorITCase
      

       

      I've spent some time to debug this case in local env, but unfortunately I didn't find the root cause. I think this is the same case with FLINK-22129, FLINK-22100, but after the debug, these failed tests in HiveTableSourceITCase should not exist bugs, and the root cause may be related to FLINK-21996.

      Given some debug results:

      • 1. Run HiveTableSourceITCase.testStreamPartitionReadByCreateTime times out repeatedly in IDE, it mostly failed at about the 36th round
      • 2. The thread usually hangs on 
      org.apache.flink.streaming.api.operators.collect.CollectResultFetcher.next(CollectResultFetcher.java:119)
      org.apache.flink.streaming.api.operators.collect.CollectResultFetcher.sendRequest(CollectResultFetcher.java:155)

      I try to add timeout for CompletableFutures in CollectResultFetcher, but looks it doesn't help

      • 3. When the thread stuck, the SourceReaderBase#addSplits may receive split or may not(I reproduced multiple rounds), but before the test failed, there's a checkpoint ERRORR which may be related to the case.
      INFO  org.apache.flink.runtime.jobmaster.JobMaster                 [] - Running initialization on master for job collect (3572e9cf6c0829ebc9f765555327388d).
      INFO  org.apache.flink.runtime.checkpoint.CheckpointCoordinator    [] - No checkpoint found during restore.
      INFO  org.apache.flink.runtime.jobmaster.JobMaster                 [] - Using failover strategy    org.apache.flink.runtime.executiongraph.failover.flip1.RestartPipelinedRegionFailoverStrategy@76f98551 for collect (3572e9cf6c0829ebc9f765555327388d).
      INFO  org.apache.flink.runtime.highavailability.nonha.embedded.EmbeddedLeaderService [] - Received confirmation of leadership for leader akka://flink/user/rpc/jobmanager_147 , session=0d15d532-d739-410f-9
      .....
       WARN  org.apache.flink.runtime.checkpoint.CheckpointCoordinator    [] - Failed to trigger checkpoint for job 3572e9cf6c0829ebc9f765555327388d.)
      org.apache.flink.runtime.checkpoint.CheckpointException: Checkpoint triggering task Source: HiveSource-source_db.stream_create_time_test -> SinkConversionToTuple2 -> Sink: Select table sink (1/1) of job 3572e9cf6c0829ebc9f765555327388d has not being executed at the moment. Aborting checkpoint. Failure reason: Not all required tasks are currently running.
              at org.apache.flink.runtime.checkpoint.DefaultCheckpointPlanCalculator.checkTasksStarted(DefaultCheckpointPlanCalculator.java:152) ~[classes/:?]
              at org.apache.flink.runtime.checkpoint.DefaultCheckpointPlanCalculator.lambda$calculateCheckpointPlan$1(DefaultCheckpointPlanCalculator.java:114) ~[classes/:?]
              at java.util.concurrent.CompletableFuture$AsyncSupply.run(CompletableFuture.java:1590) ~[?:1.8.0_172]
              at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleRunAsync(AkkaRpcActor.java:440) ~[classes/:?]
              at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleRpcMessage(AkkaRpcActor.java:208) ~[classes/:?]
              at org.apache.flink.runtime.rpc.akka.FencedAkkaRpcActor.handleRpcMessage(FencedAkkaRpcActor.java:77) ~[classes/:?]
              at org.apache.flink.runtime.rpc.akka.AkkaRpcActor.handleMessage(AkkaRpcActor.java:158) ~[classes/:?]
              at akka.japi.pf.UnitCaseStatement.apply(CaseStatements.scala:26) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.japi.pf.UnitCaseStatement.apply(CaseStatements.scala:21) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at scala.PartialFunction$class.applyOrElse(PartialFunction.scala:123) ~[scala-library-2.11.12.jar:?]
              at akka.japi.pf.UnitCaseStatement.applyOrElse(CaseStatements.scala:21) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:170) ~[scala-library-2.11.12.jar:?]
              at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:171) ~[scala-library-2.11.12.jar:?]
              at scala.PartialFunction$OrElse.applyOrElse(PartialFunction.scala:171) ~[scala-library-2.11.12.jar:?]
              at akka.actor.Actor$class.aroundReceive(Actor.scala:517) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.actor.AbstractActor.aroundReceive(AbstractActor.scala:225) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.actor.ActorCell.receiveMessage(ActorCell.scala:592) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.actor.ActorCell.invoke(ActorCell.scala:561) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.Mailbox.processMailbox(Mailbox.scala:258) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.Mailbox.run(Mailbox.scala:225) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.Mailbox.exec(Mailbox.scala:235) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.forkjoin.ForkJoinTask.doExec(ForkJoinTask.java:260) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.forkjoin.ForkJoinPool$WorkQueue.runTask(ForkJoinPool.java:1339) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.forkjoin.ForkJoinPool.runWorker(ForkJoinPool.java:1979) ~[akka-actor_2.11-2.5.21.jar:2.5.21]
              at akka.dispatch.forkjoin.ForkJoinWorkerThread.run(ForkJoinWorkerThread.java:107) ~[akka-actor_2.11-2.5.21.jar:2.5.21]

      Attachments

        Issue Links

          Activity

            People

              leonard Leonard Xu
              rmetzger Robert Metzger
              Votes:
              0 Vote for this issue
              Watchers:
              9 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved: