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 */
018package org.apache.hadoop.mapreduce;
019
020import java.io.DataInput;
021import java.io.DataOutput;
022import java.io.IOException;
023import java.util.HashMap;
024import java.util.Map;
025import java.util.Map.Entry;
026
027import org.apache.hadoop.classification.InterfaceAudience;
028import org.apache.hadoop.classification.InterfaceStability;
029import org.apache.hadoop.io.Text;
030import org.apache.hadoop.io.Writable;
031import org.apache.hadoop.io.WritableFactories;
032import org.apache.hadoop.io.WritableFactory;
033import org.apache.hadoop.io.WritableUtils;
034import org.apache.hadoop.security.authorize.AccessControlList;
035import org.apache.hadoop.util.StringInterner;
036
037/**************************************************
038 * Describes the current status of a job.
039 **************************************************/
040@InterfaceAudience.Public
041@InterfaceStability.Evolving
042public class JobStatus implements Writable, Cloneable {
043
044  static {                                      // register a ctor
045    WritableFactories.setFactory
046      (JobStatus.class,
047       new WritableFactory() {
048         public Writable newInstance() { return new JobStatus(); }
049       });
050  }
051
052  /**
053   * Current state of the job 
054   */
055  public static enum State {
056    RUNNING(1),
057    SUCCEEDED(2),
058    FAILED(3),
059    PREP(4),
060    KILLED(5);
061    
062    int value;
063    
064    State(int value) {
065      this.value = value;
066    }
067    
068    public int getValue() {
069      return value; 
070    }
071    
072  };
073  
074  private JobID jobid;
075  private float mapProgress;
076  private float reduceProgress;
077  private float cleanupProgress;
078  private float setupProgress;
079  private State runState;
080  private long startTime;
081  private String user;
082  private String queue;
083  private JobPriority priority;
084  private String schedulingInfo="NA";
085  private String failureInfo = "NA";
086
087  private Map<JobACL, AccessControlList> jobACLs =
088      new HashMap<JobACL, AccessControlList>();
089
090  private String jobName;
091  private String jobFile;
092  private long finishTime;
093  private boolean isRetired;
094  private String historyFile = "";
095  private String trackingUrl ="";
096  private int numUsedSlots;
097  private int numReservedSlots;
098  private long usedMem;
099  private long reservedMem;
100  private long neededMem;
101  private boolean isUber;
102    
103  /**
104   */
105  public JobStatus() {
106  }
107
108  /**
109   * Create a job status object for a given jobid.
110   * @param jobid The jobid of the job
111   * @param setupProgress The progress made on the setup
112   * @param mapProgress The progress made on the maps
113   * @param reduceProgress The progress made on the reduces
114   * @param cleanupProgress The progress made on the cleanup
115   * @param runState The current state of the job
116   * @param jp Priority of the job.
117   * @param user userid of the person who submitted the job.
118   * @param jobName user-specified job name.
119   * @param jobFile job configuration file.
120   * @param trackingUrl link to the web-ui for details of the job.
121   */
122   public JobStatus(JobID jobid, float setupProgress, float mapProgress,
123                    float reduceProgress, float cleanupProgress,
124                    State runState, JobPriority jp, String user, String jobName, 
125                    String jobFile, String trackingUrl) {
126     this(jobid, setupProgress, mapProgress, reduceProgress, cleanupProgress, 
127         runState, jp, user, jobName, "default", jobFile, trackingUrl, false);
128   }
129
130   /**
131    * Create a job status object for a given jobid.
132    * @param jobid The jobid of the job
133    * @param setupProgress The progress made on the setup
134    * @param mapProgress The progress made on the maps
135    * @param reduceProgress The progress made on the reduces
136    * @param cleanupProgress The progress made on the cleanup
137    * @param runState The current state of the job
138    * @param jp Priority of the job.
139    * @param user userid of the person who submitted the job.
140    * @param jobName user-specified job name.
141    * @param queue queue name
142    * @param jobFile job configuration file.
143    * @param trackingUrl link to the web-ui for details of the job.
144    */
145    public JobStatus(JobID jobid, float setupProgress, float mapProgress,
146                     float reduceProgress, float cleanupProgress,
147                     State runState, JobPriority jp,
148                     String user, String jobName, String queue,
149                     String jobFile, String trackingUrl) {
150      this(jobid, setupProgress, mapProgress, reduceProgress, cleanupProgress,
151          runState, jp, user, jobName, queue, jobFile, trackingUrl, false);
152    }
153
154   /**
155   * Create a job status object for a given jobid.
156   * @param jobid The jobid of the job
157   * @param setupProgress The progress made on the setup
158   * @param mapProgress The progress made on the maps
159   * @param reduceProgress The progress made on the reduces
160   * @param cleanupProgress The progress made on the cleanup
161   * @param runState The current state of the job
162   * @param jp Priority of the job.
163   * @param user userid of the person who submitted the job.
164   * @param jobName user-specified job name.
165   * @param queue queue name
166   * @param jobFile job configuration file.
167   * @param trackingUrl link to the web-ui for details of the job.
168   * @param isUber Whether job running in uber mode
169   */
170  public JobStatus(JobID jobid, float setupProgress, float mapProgress,
171                    float reduceProgress, float cleanupProgress,
172                    State runState, JobPriority jp,
173                    String user, String jobName, String queue,
174                    String jobFile, String trackingUrl, boolean isUber) {
175     this(jobid, setupProgress, mapProgress, reduceProgress, cleanupProgress,
176         runState, jp, user, jobName, queue, jobFile, trackingUrl, isUber, "");
177  }
178
179 /**
180  * Create a job status object for a given jobid.
181  * @param jobid The jobid of the job
182  * @param setupProgress The progress made on the setup
183  * @param mapProgress The progress made on the maps
184  * @param reduceProgress The progress made on the reduces
185  * @param cleanupProgress The progress made on the cleanup
186  * @param runState The current state of the job
187  * @param jp Priority of the job.
188  * @param user userid of the person who submitted the job.
189  * @param jobName user-specified job name.
190  * @param queue queue name
191  * @param jobFile job configuration file.
192  * @param trackingUrl link to the web-ui for details of the job.
193  * @param isUber Whether job running in uber mode
194  * @param historyFile history file
195  */
196  public JobStatus(JobID jobid, float setupProgress, float mapProgress,
197                   float reduceProgress, float cleanupProgress,
198                   State runState, JobPriority jp,
199                   String user, String jobName, String queue,
200                   String jobFile, String trackingUrl, boolean isUber,
201                   String historyFile) {
202    this.jobid = jobid;
203    this.setupProgress = setupProgress;
204    this.mapProgress = mapProgress;
205    this.reduceProgress = reduceProgress;
206    this.cleanupProgress = cleanupProgress;
207    this.runState = runState;
208    this.user = user;
209    this.queue = queue;
210    if (jp == null) {
211      throw new IllegalArgumentException("Job Priority cannot be null.");
212    }
213    priority = jp;
214    this.jobName = jobName;
215    this.jobFile = jobFile;
216    this.trackingUrl = trackingUrl;
217    this.isUber = isUber;
218    this.historyFile = historyFile;
219  }
220
221
222  /**
223   * Sets the map progress of this job
224   * @param p The value of map progress to set to
225   */
226  protected synchronized void setMapProgress(float p) { 
227    this.mapProgress = (float) Math.min(1.0, Math.max(0.0, p)); 
228  }
229
230  /**
231   * Sets the cleanup progress of this job
232   * @param p The value of cleanup progress to set to
233   */
234  protected synchronized void setCleanupProgress(float p) { 
235    this.cleanupProgress = (float) Math.min(1.0, Math.max(0.0, p)); 
236  }
237
238  /**
239   * Sets the setup progress of this job
240   * @param p The value of setup progress to set to
241   */
242  protected synchronized void setSetupProgress(float p) { 
243    this.setupProgress = (float) Math.min(1.0, Math.max(0.0, p)); 
244  }
245
246  /**
247   * Sets the reduce progress of this Job
248   * @param p The value of reduce progress to set to
249   */
250  protected synchronized void setReduceProgress(float p) { 
251    this.reduceProgress = (float) Math.min(1.0, Math.max(0.0, p)); 
252  }
253    
254  /**
255   * Set the priority of the job, defaulting to NORMAL.
256   * @param jp new job priority
257   */
258  protected synchronized void setPriority(JobPriority jp) {
259    if (jp == null) {
260      throw new IllegalArgumentException("Job priority cannot be null.");
261    }
262    priority = jp;
263  }
264  
265  /** 
266   * Set the finish time of the job
267   * @param finishTime The finishTime of the job
268   */
269  protected synchronized void setFinishTime(long finishTime) {
270    this.finishTime = finishTime;
271  }
272
273  /**
274   * Set the job history file url for a completed job
275   */
276  protected synchronized void setHistoryFile(String historyFile) {
277    this.historyFile = historyFile;
278  }
279
280  /**
281   * Set the link to the web-ui for details of the job.
282   */
283  protected synchronized void setTrackingUrl(String trackingUrl) {
284    this.trackingUrl = trackingUrl;
285  }
286
287  /**
288   * Set the job retire flag to true.
289   */
290  protected synchronized void setRetired() {
291    this.isRetired = true;
292  }
293
294  /**
295   * Change the current run state of the job.
296   */
297  protected synchronized void setState(State state) {
298    this.runState = state;
299  }
300
301  /** 
302   * Set the start time of the job
303   * @param startTime The startTime of the job
304   */
305  protected synchronized void setStartTime(long startTime) { 
306    this.startTime = startTime;
307  }
308    
309  /**
310   * @param userName The username of the job
311   */
312  protected synchronized void setUsername(String userName) { 
313    this.user = userName;
314  }
315
316  /**
317   * Used to set the scheduling information associated to a particular Job.
318   * 
319   * @param schedulingInfo Scheduling information of the job
320   */
321  protected synchronized void setSchedulingInfo(String schedulingInfo) {
322    this.schedulingInfo = schedulingInfo;
323  }
324
325  /**
326   * Set the job acls.
327   * 
328   * @param acls {@link Map} from {@link JobACL} to {@link AccessControlList}
329   */
330  protected synchronized void setJobACLs(Map<JobACL, AccessControlList> acls) {
331    this.jobACLs = acls;
332  }
333
334  /**
335   * Set queue name
336   * @param queue queue name
337   */
338  protected synchronized void setQueue(String queue) {
339    this.queue = queue;
340  }
341
342  /**
343   * Set diagnostic information.
344   * @param failureInfo diagnostic information
345   */
346  protected synchronized void setFailureInfo(String failureInfo) {
347    this.failureInfo = failureInfo;
348  }
349  
350  /**
351   * Get queue name
352   * @return queue name
353   */
354  public synchronized String getQueue() {
355    return queue;
356  }
357
358  /**
359   * @return Percentage of progress in maps 
360   */
361  public synchronized float getMapProgress() { return mapProgress; }
362    
363  /**
364   * @return Percentage of progress in cleanup 
365   */
366  public synchronized float getCleanupProgress() { return cleanupProgress; }
367    
368  /**
369   * @return Percentage of progress in setup 
370   */
371  public synchronized float getSetupProgress() { return setupProgress; }
372    
373  /**
374   * @return Percentage of progress in reduce 
375   */
376  public synchronized float getReduceProgress() { return reduceProgress; }
377    
378  /**
379   * @return running state of the job
380   */
381  public synchronized State getState() { return runState; }
382    
383  /**
384   * @return start time of the job
385   */
386  synchronized public long getStartTime() { return startTime;}
387
388  @Override
389  public Object clone() {
390    try {
391      return super.clone();
392    } catch (CloneNotSupportedException cnse) {
393      // Shouldn't happen since we do implement Clonable
394      throw new InternalError(cnse.toString());
395    }
396  }
397  
398  /**
399   * @return The jobid of the Job
400   */
401  public JobID getJobID() { return jobid; }
402    
403  /**
404   * @return the username of the job
405   */
406  public synchronized String getUsername() { return this.user;}
407  
408  /**
409   * Gets the Scheduling information associated to a particular Job.
410   * @return the scheduling information of the job
411   */
412  public synchronized String getSchedulingInfo() {
413   return schedulingInfo;
414  }
415
416  /**
417   * Get the job acls.
418   * 
419   * @return a {@link Map} from {@link JobACL} to {@link AccessControlList}
420   */
421  public synchronized Map<JobACL, AccessControlList> getJobACLs() {
422    return jobACLs;
423  }
424
425  /**
426   * Return the priority of the job
427   * @return job priority
428   */
429   public synchronized JobPriority getPriority() { return priority; }
430  
431   /**
432    * Gets any available info on the reason of failure of the job.
433    * @return diagnostic information on why a job might have failed.
434    */
435   public synchronized String getFailureInfo() {
436     return this.failureInfo;
437   }
438
439
440  /**
441   * Returns true if the status is for a completed job.
442   */
443  public synchronized boolean isJobComplete() {
444    return (runState == JobStatus.State.SUCCEEDED || 
445            runState == JobStatus.State.FAILED || 
446            runState == JobStatus.State.KILLED);
447  }
448
449  ///////////////////////////////////////
450  // Writable
451  ///////////////////////////////////////
452  public synchronized void write(DataOutput out) throws IOException {
453    jobid.write(out);
454    out.writeFloat(setupProgress);
455    out.writeFloat(mapProgress);
456    out.writeFloat(reduceProgress);
457    out.writeFloat(cleanupProgress);
458    WritableUtils.writeEnum(out, runState);
459    out.writeLong(startTime);
460    Text.writeString(out, user);
461    WritableUtils.writeEnum(out, priority);
462    Text.writeString(out, schedulingInfo);
463    out.writeLong(finishTime);
464    out.writeBoolean(isRetired);
465    Text.writeString(out, historyFile);
466    Text.writeString(out, jobName);
467    Text.writeString(out, trackingUrl);
468    Text.writeString(out, jobFile);
469    out.writeBoolean(isUber);
470
471    // Serialize the job's ACLs
472    out.writeInt(jobACLs.size());
473    for (Entry<JobACL, AccessControlList> entry : jobACLs.entrySet()) {
474      WritableUtils.writeEnum(out, entry.getKey());
475      entry.getValue().write(out);
476    }
477  }
478
479  public synchronized void readFields(DataInput in) throws IOException {
480    this.jobid = new JobID();
481    this.jobid.readFields(in);
482    this.setupProgress = in.readFloat();
483    this.mapProgress = in.readFloat();
484    this.reduceProgress = in.readFloat();
485    this.cleanupProgress = in.readFloat();
486    this.runState = WritableUtils.readEnum(in, State.class);
487    this.startTime = in.readLong();
488    this.user = StringInterner.weakIntern(Text.readString(in));
489    this.priority = WritableUtils.readEnum(in, JobPriority.class);
490    this.schedulingInfo = StringInterner.weakIntern(Text.readString(in));
491    this.finishTime = in.readLong();
492    this.isRetired = in.readBoolean();
493    this.historyFile = StringInterner.weakIntern(Text.readString(in));
494    this.jobName = StringInterner.weakIntern(Text.readString(in));
495    this.trackingUrl = StringInterner.weakIntern(Text.readString(in));
496    this.jobFile = StringInterner.weakIntern(Text.readString(in));
497    this.isUber = in.readBoolean();
498
499    // De-serialize the job's ACLs
500    int numACLs = in.readInt();
501    for (int i = 0; i < numACLs; i++) {
502      JobACL aclType = WritableUtils.readEnum(in, JobACL.class);
503      AccessControlList acl = new AccessControlList(" ");
504      acl.readFields(in);
505      this.jobACLs.put(aclType, acl);
506    }
507  }
508
509  /**
510   * Get the user-specified job name.
511   */
512  public String getJobName() {
513    return jobName;
514  }
515
516  /**
517   * Get the configuration file for the job.
518   */
519  public String getJobFile() {
520    return jobFile;
521  }
522
523  /**
524   * Get the link to the web-ui for details of the job.
525   */
526  public synchronized String getTrackingUrl() {
527    return trackingUrl;
528  }
529
530  /**
531   * Get the finish time of the job.
532   */
533  public synchronized long getFinishTime() { 
534    return finishTime;
535  }
536
537  /**
538   * Check whether the job has retired.
539   */
540  public synchronized boolean isRetired() {
541    return isRetired;
542  }
543
544  /**
545   * @return the job history file name for a completed job. If job is not 
546   * completed or history file not available then return null.
547   */
548  public synchronized String getHistoryFile() {
549    return historyFile;
550  }
551
552  /**
553   * @return number of used mapred slots
554   */
555  public int getNumUsedSlots() {
556    return numUsedSlots;
557  }
558
559  /**
560   * @param n number of used mapred slots
561   */
562  public void setNumUsedSlots(int n) {
563    numUsedSlots = n;
564  }
565
566  /**
567   * @return the number of reserved slots
568   */
569  public int getNumReservedSlots() {
570    return numReservedSlots;
571  }
572
573  /**
574   * @param n the number of reserved slots
575   */
576  public void setNumReservedSlots(int n) {
577    this.numReservedSlots = n;
578  }
579
580  /**
581   * @return the used memory
582   */
583  public long getUsedMem() {
584    return usedMem;
585  }
586
587  /**
588   * @param m the used memory
589   */
590  public void setUsedMem(long m) {
591    this.usedMem = m;
592  }
593
594  /**
595   * @return the reserved memory
596   */
597  public long getReservedMem() {
598    return reservedMem;
599 }
600
601  /**
602   * @param r the reserved memory
603   */
604  public void setReservedMem(long r) {
605    this.reservedMem = r;
606  }
607
608  /**
609   * @return the needed memory
610   */
611  public long getNeededMem() {
612  return neededMem;
613 }
614
615  /**
616   * @param n the needed memory
617   */
618  public void setNeededMem(long n) {
619    this.neededMem = n;
620  }
621
622  /**
623   * Whether job running in uber mode
624   * @return job in uber-mode
625   */
626  public synchronized boolean isUber() {
627    return isUber;
628  }
629  
630  /**
631   * Set uber-mode flag 
632   * @param isUber Whether job running in uber-mode
633   */
634  public synchronized void setUber(boolean isUber) {
635    this.isUber = isUber;
636  }
637  
638  public String toString() {
639    StringBuffer buffer = new StringBuffer();
640    buffer.append("job-id : " + jobid);
641    buffer.append("uber-mode : " + isUber);
642    buffer.append("map-progress : " + mapProgress);
643    buffer.append("reduce-progress : " + reduceProgress);
644    buffer.append("cleanup-progress : " + cleanupProgress);
645    buffer.append("setup-progress : " + setupProgress);
646    buffer.append("runstate : " + runState);
647    buffer.append("start-time : " + startTime);
648    buffer.append("user-name : " + user);
649    buffer.append("priority : " + priority);
650    buffer.append("scheduling-info : " + schedulingInfo);
651    buffer.append("num-used-slots" + numUsedSlots);
652    buffer.append("num-reserved-slots" + numReservedSlots);
653    buffer.append("used-mem" + usedMem);
654    buffer.append("reserved-mem" + reservedMem);
655    buffer.append("needed-mem" + neededMem);
656    return buffer.toString();
657  }
658}