Skip to content

Commit

Permalink
SPARK-5613: Catch the ApplicationNotFoundException exception to avoid…
Browse files Browse the repository at this point in the history
… thread from getting killed on yarn restart.

[SPARK-5613] Added a  catch block to catch the ApplicationNotFoundException. Without this catch block the thread gets killed on occurrence of this exception. This Exception occurs when yarn restarts and tries to find an application id for a spark job which got interrupted due to yarn getting stopped.
See the stacktrace in the bug for more details.

Author: Kashish Jain <kashish.jain@guavus.com>

Closes #4392 from kasjain/branch-1.2 and squashes the following commits:

4831000 [Kashish Jain] SPARK-5613: Catch the ApplicationNotFoundException exception to avoid thread from getting killed on yarn restart.
  • Loading branch information
kasjain authored and Andrew Or committed Feb 6, 2015
1 parent 36f70de commit d89964f
Showing 1 changed file with 9 additions and 2 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -20,6 +20,7 @@ package org.apache.spark.scheduler.cluster
import scala.collection.mutable.ArrayBuffer

import org.apache.hadoop.yarn.api.records.{ApplicationId, YarnApplicationState}
import org.apache.hadoop.yarn.exceptions.ApplicationNotFoundException

import org.apache.spark.{SparkException, Logging, SparkContext}
import org.apache.spark.deploy.yarn.{Client, ClientArguments}
Expand Down Expand Up @@ -132,8 +133,14 @@ private[spark] class YarnClientSchedulerBackend(
val t = new Thread {
override def run() {
while (!stopping) {
val report = client.getApplicationReport(appId)
val state = report.getYarnApplicationState()
var state : YarnApplicationState = null
try {
val report = client.getApplicationReport(appId)
state = report.getYarnApplicationState()
} catch {
case e : ApplicationNotFoundException =>
state = YarnApplicationState.KILLED
}
if (state == YarnApplicationState.FINISHED ||
state == YarnApplicationState.KILLED ||
state == YarnApplicationState.FAILED) {
Expand Down

0 comments on commit d89964f

Please sign in to comment.