[
https://issues.apache.org/jira/browse/FLINK-5476?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15861388#comment-15861388
]
Dmitrii Kniazev commented on FLINK-5476:
----------------------------------------
For solving this task i decide invoke method
StandaloneClusterClient#getClusterStatus()
inside StandaloneClusterClient#waitForClusterToBeReady() to check cluster
availability.
But it causes to fall of flink cluster (v1.3) with following error:
{panel:title=Log}
2017-02-10 15:35:25,460 INFO
org.apache.flink.runtime.clusterframework.standalone.StandaloneResourceManager
- Consolidated 1 TaskManagers
2017-02-10 15:36:16,588 ERROR akka.actor.OneForOneStrategy
- GetClusterStatus (of class
org.apache.flink.runtime.clusterframework.messages.GetClusterStatus)
scala.MatchError: GetClusterStatus (of class
org.apache.flink.runtime.clusterframework.messages.GetClusterStatus)
at scala.PartialFunction$$anon$1.apply(PartialFunction.scala:248)
at scala.PartialFunction$$anon$1.apply(PartialFunction.scala:246)
at
org.apache.flink.runtime.jobmanager.JobManager$$anonfun$handleMessage$1.applyOrElse(JobManager.scala:290)
at
scala.runtime.AbstractPartialFunction$mcVL$sp.apply$mcVL$sp(AbstractPartialFunction.scala:33)
at
scala.runtime.AbstractPartialFunction$mcVL$sp.apply(AbstractPartialFunction.scala:33)
at
scala.runtime.AbstractPartialFunction$mcVL$sp.apply(AbstractPartialFunction.scala:25)
at
org.apache.flink.runtime.LeaderSessionMessageFilter$$anonfun$receive$1.applyOrElse(LeaderSessionMessageFilter.scala:44)
at
scala.runtime.AbstractPartialFunction$mcVL$sp.apply$mcVL$sp(AbstractPartialFunction.scala:33)
at
scala.runtime.AbstractPartialFunction$mcVL$sp.apply(AbstractPartialFunction.scala:33)
at
scala.runtime.AbstractPartialFunction$mcVL$sp.apply(AbstractPartialFunction.scala:25)
at
org.apache.flink.runtime.LogMessages$$anon$1.apply(LogMessages.scala:33)
at
org.apache.flink.runtime.LogMessages$$anon$1.apply(LogMessages.scala:28)
at scala.PartialFunction$class.applyOrElse(PartialFunction.scala:118)
at
org.apache.flink.runtime.LogMessages$$anon$1.applyOrElse(LogMessages.scala:28)
at akka.actor.Actor$class.aroundReceive(Actor.scala:467)
at
org.apache.flink.runtime.jobmanager.JobManager.aroundReceive(JobManager.scala:118)
at akka.actor.ActorCell.receiveMessage(ActorCell.scala:516)
at akka.actor.ActorCell.invoke(ActorCell.scala:487)
at akka.dispatch.Mailbox.processMailbox(Mailbox.scala:238)
at akka.dispatch.Mailbox.run(Mailbox.scala:220)
at
akka.dispatch.ForkJoinExecutorConfigurator$AkkaForkJoinTask.exec(AbstractDispatcher.scala:397)
at scala.concurrent.forkjoin.ForkJoinTask.doExec(ForkJoinTask.java:260)
at
scala.concurrent.forkjoin.ForkJoinPool$WorkQueue.runTask(ForkJoinPool.java:1339)
at
scala.concurrent.forkjoin.ForkJoinPool.runWorker(ForkJoinPool.java:1979)
at
scala.concurrent.forkjoin.ForkJoinWorkerThread.run(ForkJoinWorkerThread.java:107)
2017-02-10 15:36:16,593 INFO org.apache.flink.runtime.jobmanager.JobManager
- Stopping JobManager
akka.tcp://flink@localhost:6123/user/jobmanager.
2017-02-10 15:36:16,599 INFO org.apache.flink.runtime.taskmanager.TaskManager
- TaskManager akka://flink/user/taskmanager disconnects from
JobManager akka://flink/user/jobmanager: JobManager requested disconnect:
JobManager is shuttind down.
2017-02-10 15:36:16,599 INFO org.apache.flink.runtime.taskmanager.TaskManager
- Disassociating from JobManager
2017-02-10 15:36:16,600 INFO org.apache.flink.runtime.blob.BlobServer
- Stopped BLOB server at 0.0.0.0:46780
2017-02-10 15:36:16,603 ERROR org.apache.flink.runtime.jobmanager.JobManager
- Actor akka://flink/user/jobmanager#-364585011 terminated,
stopping process...
2017-02-10 15:36:16,603 INFO org.apache.flink.runtime.blob.BlobCache
- Shutting down BlobCache
2017-02-10 15:36:16,605 INFO org.apache.flink.runtime.taskmanager.TaskManager
- Trying to register at JobManager
akka.tcp://flink@localhost:6123/user/jobmanager (attempt 1, timeout: 500
milliseconds)
2017-02-10 15:36:16,708 INFO
org.apache.flink.runtime.webmonitor.WebRuntimeMonitor - Removing web
dashboard root cache directory
/tmp/flink-web-ad368bef-7394-4065-8c98-704fb94777b6
2017-02-10 15:36:16,714 INFO
org.apache.flink.runtime.webmonitor.WebRuntimeMonitor - Removing web
dashboard jar upload directory
/tmp/flink-web-b0f08882-207c-49cc-bfef-30badbfab011
2017-02-10 15:36:16,722 INFO
org.apache.flink.runtime.io.disk.iomanager.IOManager - I/O manager
removed spill file directory /tmp/flink-io-387025c9-b52c-4b71-9122-8d8d96c5a8a6
{panel}
I think it a bug. What do you think about it and about proposed decision of
task?
> Fail fast if trying to submit a job to a non-existing Flink cluster
> -------------------------------------------------------------------
>
> Key: FLINK-5476
> URL: https://issues.apache.org/jira/browse/FLINK-5476
> Project: Flink
> Issue Type: Improvement
> Components: Client
> Affects Versions: 1.2.0, 1.3.0
> Reporter: Till Rohrmann
> Assignee: Dmitrii Kniazev
> Priority: Minor
>
> In case of entering the wrong job manager address when submitting a job via
> {{flink run}}, the {{JobClientActor}} waits per default {{60 s}} until a
> {{JobClientActorConnectionException}}, indicating that the {{JobManager}} is
> no longer reachable, is thrown. In order to fail fast in case of wrong
> connection information, we could change it such that it uses initially a much
> lower timeout and only increases the timeout if it had at least once
> successfully connected to a {{JobManager}} before.
--
This message was sent by Atlassian JIRA
(v6.3.15#6346)