Uploaded image for project: 'Hadoop Map/Reduce'
  1. Hadoop Map/Reduce
  2. MAPREDUCE-5422 [Umbrella] Fix invalid state transitions in MRAppMaster
  3. MAPREDUCE-5400

MRAppMaster throws InvalidStateTransitonException: Invalid event: JOB_TASK_COMPLETED at SUCCEEDED for JobImpl

    XMLWordPrintableJSON

Details

    • Sub-task
    • Status: Resolved
    • Minor
    • Resolution: Cannot Reproduce
    • 2.0.5-alpha
    • None
    • applicationmaster
    • None

    Description

      Step 1: Install cluster with HDFS , MR
      Step 2: Execute a job
      Step 3: Issue a kill task attempt for which the task has got completed.

      Rex@HOST-10-18-91-55:~/NodeAgentTmpDir/installations/hadoop-2.0.5.tar/hadoop-2.0.5/bin> ./mapred job -kill-task attempt_1373875322959_0032_m_000000_0
      No GC_PROFILE is given. Defaults to medium.
      13/07/15 14:46:32 INFO service.AbstractService: Service:org.apache.hadoop.yarn.client.YarnClientImpl is inited.
      13/07/15 14:46:32 INFO proxy.ResourceManagerProxies: HA Proxy Creation with xface : interface org.apache.hadoop.yarn.api.ClientRMProtocol
      13/07/15 14:46:33 INFO service.AbstractService: Service:org.apache.hadoop.yarn.client.YarnClientImpl is started.
      Killed task attempt_1373875322959_0032_m_000000_0

      Observation:
      ===========
      1. task state has been transitioned from SUCCEEDED to SCHEDULED
      2. For a Succeeded attempt , when client issues Kill , then the client is notified as killed for a succeeded attempt.
      3. Launched second task_attempt which is succeeded and then killed later on client request.
      4. Even after the job state transitioned from SUCCEEDED to ERROR , on UI the state is succeeded

      Issue :
      =====
      1. Client has been notified that the atttempt is killed , but acutually the attempt is succeeded and the same is displayed in JHS UI.
      2. At App master InvalidStateTransitonException is thrown .
      3. At client side and JHS job has exited with state Finished/succeeded ,At RM side the state is Finished/Failed.

      AM Logs:
      ========
      2013-07-15 14:46:25,461 INFO [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.TaskAttemptImpl: attempt_1373875322959_0032_m_000000_0 TaskAttempt Transitioned from RUNNING to SUCCEEDED
      2013-07-15 14:46:25,468 INFO [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.TaskImpl: Task succeeded with attempt attempt_1373875322959_0032_m_000000_0
      2013-07-15 14:46:25,470 INFO [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.TaskImpl: task_1373875322959_0032_m_000000 Task Transitioned from RUNNING to SUCCEEDED
      2013-07-15 14:46:33,810 INFO [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.TaskImpl: task_1373875322959_0032_m_000000 Task Transitioned from SUCCEEDED to SCHEDULED
      2013-07-15 14:46:37,344 INFO [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.TaskImpl: Task succeeded with attempt attempt_1373875322959_0032_m_000000_1
      2013-07-15 14:46:37,344 INFO [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.TaskImpl: task_1373875322959_0032_m_000000 Task Transitioned from RUNNING to SUCCEEDED
      2013-07-15 14:46:37,345 ERROR [AsyncDispatcher event handler] org.apache.hadoop.mapreduce.v2.app.job.impl.JobImpl: Can't handle this event at current state
      org.apache.hadoop.yarn.state.InvalidStateTransitonException: Invalid event: JOB_TASK_COMPLETED at SUCCEEDED
      at org.apache.hadoop.yarn.state.StateMachineFactory.doTransition(StateMachineFactory.java:302)
      at org.apache.hadoop.yarn.state.StateMachineFactory.access$300(StateMachineFactory.java:43)
      at org.apache.hadoop.yarn.state.StateMachineFactory$InternalStateMachine.doTransition(StateMachineFactory.java:445)
      at org.apache.hadoop.mapreduce.v2.app.job.impl.JobImpl.handle(JobImpl.java:866)
      at org.apache.hadoop.mapreduce.v2.app.job.impl.JobImpl.handle(JobImpl.java:128)
      at org.apache.hadoop.mapreduce.v2.app.MRAppMaster$JobEventDispatcher.handle(MRAppMaster.java:1095)
      at org.apache.hadoop.mapreduce.v2.app.MRAppMaster$JobEventDispatcher.handle(MRAppMaster.java:1091)
      at org.apache.hadoop.yarn.event.AsyncDispatcher.dispatch(AsyncDispatcher.java:130)
      at org.apache.hadoop.yarn.event.AsyncDispatcher$1.run(AsyncDispatcher.java:77)
      at java.lang.Thread.run(Thread.java:662)

      Attachments

        1. MAPREDUCE-5400-1.patch
          2 kB
          Devaraj Kavali
        2. MAPREDUCE-5400.patch
          2 kB
          Devaraj Kavali

        Issue Links

          Activity

            People

              devaraj Devaraj Kavali
              andreina J.Andreina
              Votes:
              0 Vote for this issue
              Watchers:
              9 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved: