001    /**
002     * Licensed to the Apache Software Foundation (ASF) under one
003     * or more contributor license agreements.  See the NOTICE file
004     * distributed with this work for additional information
005     * regarding copyright ownership.  The ASF licenses this file
006     * to you under the Apache License, Version 2.0 (the
007     * "License"); you may not use this file except in compliance
008     * with the License.  You may obtain a copy of the License at
009     *
010     *     http://www.apache.org/licenses/LICENSE-2.0
011     *
012     * Unless required by applicable law or agreed to in writing, software
013     * distributed under the License is distributed on an "AS IS" BASIS,
014     * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015     * See the License for the specific language governing permissions and
016     * limitations under the License.
017     */
018    
019    package org.apache.hadoop.mapred.jobcontrol;
020    
021    
022    import java.io.IOException;
023    import java.util.ArrayList;
024    import java.util.List;
025    
026    import org.apache.commons.logging.Log;
027    import org.apache.commons.logging.LogFactory;
028    import org.apache.hadoop.classification.InterfaceAudience;
029    import org.apache.hadoop.classification.InterfaceStability;
030    import org.apache.hadoop.mapred.JobClient;
031    import org.apache.hadoop.mapred.JobConf;
032    import org.apache.hadoop.mapred.JobID;
033    import org.apache.hadoop.mapreduce.lib.jobcontrol.ControlledJob;
034    
035    @InterfaceAudience.Public
036    @InterfaceStability.Stable
037    public class Job extends ControlledJob {
038      static final Log LOG = LogFactory.getLog(Job.class);
039    
040      final public static int SUCCESS = 0;
041      final public static int WAITING = 1;
042      final public static int RUNNING = 2;
043      final public static int READY = 3;
044      final public static int FAILED = 4;
045      final public static int DEPENDENT_FAILED = 5;
046    
047      /** 
048       * Construct a job.
049       * @param jobConf a mapred job configuration representing a job to be executed.
050       * @param dependingJobs an array of jobs the current job depends on
051       */
052      @SuppressWarnings("unchecked")
053      public Job(JobConf jobConf, ArrayList<?> dependingJobs) throws IOException {
054        super(new org.apache.hadoop.mapreduce.Job(jobConf), 
055              (List<ControlledJob>) dependingJobs);
056      }
057    
058      public Job(JobConf conf) throws IOException {
059        super(conf);
060      }
061    
062      /**
063       * @return the mapred ID of this job as assigned by the mapred framework.
064       */
065      public JobID getAssignedJobID() {
066        org.apache.hadoop.mapreduce.JobID temp = super.getMapredJobId();
067        if (temp == null) {
068          return null;
069        }
070        return JobID.downgrade(temp);
071      }
072    
073      /**
074       * @deprecated setAssignedJobID should not be called.
075       * JOBID is set by the framework.
076       */
077      @Deprecated
078      public void setAssignedJobID(JobID mapredJobID) {
079        // do nothing
080      }
081    
082      /**
083       * @return the mapred job conf of this job
084       */
085      public synchronized JobConf getJobConf() {
086        return new JobConf(super.getJob().getConfiguration());
087      }
088    
089    
090      /**
091       * Set the mapred job conf for this job.
092       * @param jobConf the mapred job conf for this job.
093       */
094      public synchronized void setJobConf(JobConf jobConf) {
095        try {
096          super.setJob(new org.apache.hadoop.mapreduce.Job(jobConf));
097        } catch (IOException ioe) { 
098          LOG.info("Exception" + ioe);
099        }
100      }
101    
102      /**
103       * @return the state of this job
104       */
105      public synchronized int getState() {
106        State state = super.getJobState();
107        if (state == State.SUCCESS) {
108          return SUCCESS;
109        } 
110        if (state == State.WAITING) {
111          return WAITING;
112        }
113        if (state == State.RUNNING) {
114          return RUNNING;
115        }
116        if (state == State.READY) {
117          return READY;
118        }
119        if (state == State.FAILED ) {
120          return FAILED;
121        }
122        if (state == State.DEPENDENT_FAILED ) {
123          return DEPENDENT_FAILED;
124        }
125        return -1;
126      }
127      
128      /**
129       * This is a no-op function, Its a behavior change from 1.x We no more can
130       * change the state from job
131       * 
132       * @param state
133       *          the new state for this job.
134       */
135      @Deprecated
136      protected synchronized void setState(int state) {
137        // No-Op, we dont want to change the sate
138      }
139      
140      /**
141       * Add a job to this jobs' dependency list. 
142       * Dependent jobs can only be added while a Job 
143       * is waiting to run, not during or afterwards.
144       * 
145       * @param dependingJob Job that this Job depends on.
146       * @return <tt>true</tt> if the Job was added.
147       */
148      public synchronized boolean addDependingJob(Job dependingJob) {
149        return super.addDependingJob(dependingJob);
150      }
151      
152      /**
153       * @return the job client of this job
154       */
155      public JobClient getJobClient() {
156        try {
157          return new JobClient(super.getJob().getConfiguration());
158        } catch (IOException ioe) {
159          return null;
160        }
161      }
162    
163      /**
164       * @return the depending jobs of this job
165       */
166      public ArrayList<Job> getDependingJobs() {
167        return JobControl.castToJobList(super.getDependentJobs());
168      }
169    
170      /**
171       * @return the mapred ID of this job as assigned by the mapred framework.
172       */
173      public synchronized String getMapredJobID() {
174        if (super.getMapredJobId() != null) {
175          return super.getMapredJobId().toString();
176        }
177        return null;
178      }
179    
180      /**
181       * This is no-op method for backward compatibility. It's a behavior change
182       * from 1.x, we can not change job ids from job.
183       * 
184       * @param mapredJobID
185       *          the mapred job ID for this job.
186       */
187      @Deprecated
188      public synchronized void setMapredJobID(String mapredJobID) {
189        setAssignedJobID(JobID.forName(mapredJobID));
190      }
191    }