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.tools;
019
020import java.io.IOException;
021import java.io.OutputStreamWriter;
022import java.io.PrintWriter;
023import java.util.ArrayList;
024import java.util.Arrays;
025import java.util.HashSet;
026import java.util.List;
027import java.util.Set;
028
029import com.google.common.annotations.VisibleForTesting;
030import org.apache.commons.lang.StringUtils;
031import org.apache.commons.logging.Log;
032import org.apache.commons.logging.LogFactory;
033import org.apache.hadoop.classification.InterfaceAudience;
034import org.apache.hadoop.classification.InterfaceStability;
035import org.apache.hadoop.classification.InterfaceAudience.Private;
036import org.apache.hadoop.conf.Configuration;
037import org.apache.hadoop.conf.Configured;
038import org.apache.hadoop.ipc.RemoteException;
039import org.apache.hadoop.mapred.JobConf;
040import org.apache.hadoop.mapred.TIPStatus;
041import org.apache.hadoop.mapreduce.Cluster;
042import org.apache.hadoop.mapreduce.Counters;
043import org.apache.hadoop.mapreduce.Job;
044import org.apache.hadoop.mapreduce.JobID;
045import org.apache.hadoop.mapreduce.JobPriority;
046import org.apache.hadoop.mapreduce.JobStatus;
047import org.apache.hadoop.mapreduce.MRJobConfig;
048import org.apache.hadoop.mapreduce.TaskAttemptID;
049import org.apache.hadoop.mapreduce.TaskCompletionEvent;
050import org.apache.hadoop.mapreduce.TaskReport;
051import org.apache.hadoop.mapreduce.TaskTrackerInfo;
052import org.apache.hadoop.mapreduce.TaskType;
053import org.apache.hadoop.mapreduce.jobhistory.HistoryViewer;
054import org.apache.hadoop.mapreduce.v2.LogParams;
055import org.apache.hadoop.security.AccessControlException;
056import org.apache.hadoop.util.ExitUtil;
057import org.apache.hadoop.util.Tool;
058import org.apache.hadoop.util.ToolRunner;
059import org.apache.hadoop.yarn.logaggregation.LogCLIHelpers;
060
061import com.google.common.base.Charsets;
062
063/**
064 * Interprets the map reduce cli options 
065 */
066@InterfaceAudience.Public
067@InterfaceStability.Stable
068public class CLI extends Configured implements Tool {
069  private static final Log LOG = LogFactory.getLog(CLI.class);
070  protected Cluster cluster;
071  private static final Set<String> taskTypes = new HashSet<String>(
072      Arrays.asList("MAP", "REDUCE"));
073  private final Set<String> taskStates = new HashSet<String>(Arrays.asList(
074      "running", "completed", "pending", "failed", "killed"));
075 
076  public CLI() {
077  }
078  
079  public CLI(Configuration conf) {
080    setConf(conf);
081  }
082  
083  public int run(String[] argv) throws Exception {
084    int exitCode = -1;
085    if (argv.length < 1) {
086      displayUsage("");
087      return exitCode;
088    }    
089    // process arguments
090    String cmd = argv[0];
091    String submitJobFile = null;
092    String jobid = null;
093    String taskid = null;
094    String historyFile = null;
095    String counterGroupName = null;
096    String counterName = null;
097    JobPriority jp = null;
098    String taskType = null;
099    String taskState = null;
100    int fromEvent = 0;
101    int nEvents = 0;
102    int jpvalue = 0;
103    boolean getStatus = false;
104    boolean getCounter = false;
105    boolean killJob = false;
106    boolean listEvents = false;
107    boolean viewHistory = false;
108    boolean viewAllHistory = false;
109    boolean listJobs = false;
110    boolean listAllJobs = false;
111    boolean listActiveTrackers = false;
112    boolean listBlacklistedTrackers = false;
113    boolean displayTasks = false;
114    boolean killTask = false;
115    boolean failTask = false;
116    boolean setJobPriority = false;
117    boolean logs = false;
118
119    if ("-submit".equals(cmd)) {
120      if (argv.length != 2) {
121        displayUsage(cmd);
122        return exitCode;
123      }
124      submitJobFile = argv[1];
125    } else if ("-status".equals(cmd)) {
126      if (argv.length != 2) {
127        displayUsage(cmd);
128        return exitCode;
129      }
130      jobid = argv[1];
131      getStatus = true;
132    } else if("-counter".equals(cmd)) {
133      if (argv.length != 4) {
134        displayUsage(cmd);
135        return exitCode;
136      }
137      getCounter = true;
138      jobid = argv[1];
139      counterGroupName = argv[2];
140      counterName = argv[3];
141    } else if ("-kill".equals(cmd)) {
142      if (argv.length != 2) {
143        displayUsage(cmd);
144        return exitCode;
145      }
146      jobid = argv[1];
147      killJob = true;
148    } else if ("-set-priority".equals(cmd)) {
149      if (argv.length != 3) {
150        displayUsage(cmd);
151        return exitCode;
152      }
153      jobid = argv[1];
154      try {
155        jp = JobPriority.valueOf(argv[2]);
156      } catch (IllegalArgumentException iae) {
157        try {
158          jpvalue = Integer.parseInt(argv[2]);
159        } catch (NumberFormatException ne) {
160          LOG.info(ne);
161          displayUsage(cmd);
162          return exitCode;
163        }
164      }
165      setJobPriority = true; 
166    } else if ("-events".equals(cmd)) {
167      if (argv.length != 4) {
168        displayUsage(cmd);
169        return exitCode;
170      }
171      jobid = argv[1];
172      fromEvent = Integer.parseInt(argv[2]);
173      nEvents = Integer.parseInt(argv[3]);
174      listEvents = true;
175    } else if ("-history".equals(cmd)) {
176      if (argv.length != 2 && !(argv.length == 3 && "all".equals(argv[1]))) {
177         displayUsage(cmd);
178         return exitCode;
179      }
180      viewHistory = true;
181      if (argv.length == 3 && "all".equals(argv[1])) {
182        viewAllHistory = true;
183        historyFile = argv[2];
184      } else {
185        historyFile = argv[1];
186      }
187    } else if ("-list".equals(cmd)) {
188      if (argv.length != 1 && !(argv.length == 2 && "all".equals(argv[1]))) {
189        displayUsage(cmd);
190        return exitCode;
191      }
192      if (argv.length == 2 && "all".equals(argv[1])) {
193        listAllJobs = true;
194      } else {
195        listJobs = true;
196      }
197    } else if("-kill-task".equals(cmd)) {
198      if (argv.length != 2) {
199        displayUsage(cmd);
200        return exitCode;
201      }
202      killTask = true;
203      taskid = argv[1];
204    } else if("-fail-task".equals(cmd)) {
205      if (argv.length != 2) {
206        displayUsage(cmd);
207        return exitCode;
208      }
209      failTask = true;
210      taskid = argv[1];
211    } else if ("-list-active-trackers".equals(cmd)) {
212      if (argv.length != 1) {
213        displayUsage(cmd);
214        return exitCode;
215      }
216      listActiveTrackers = true;
217    } else if ("-list-blacklisted-trackers".equals(cmd)) {
218      if (argv.length != 1) {
219        displayUsage(cmd);
220        return exitCode;
221      }
222      listBlacklistedTrackers = true;
223    } else if ("-list-attempt-ids".equals(cmd)) {
224      if (argv.length != 4) {
225        displayUsage(cmd);
226        return exitCode;
227      }
228      jobid = argv[1];
229      taskType = argv[2];
230      taskState = argv[3];
231      displayTasks = true;
232      if (!taskTypes.contains(
233          org.apache.hadoop.util.StringUtils.toUpperCase(taskType))) {
234        System.out.println("Error: Invalid task-type: " + taskType);
235        displayUsage(cmd);
236        return exitCode;
237      }
238      if (!taskStates.contains(
239          org.apache.hadoop.util.StringUtils.toLowerCase(taskState))) {
240        System.out.println("Error: Invalid task-state: " + taskState);
241        displayUsage(cmd);
242        return exitCode;
243      }
244    } else if ("-logs".equals(cmd)) {
245      if (argv.length == 2 || argv.length ==3) {
246        logs = true;
247        jobid = argv[1];
248        if (argv.length == 3) {
249          taskid = argv[2];
250        }  else {
251          taskid = null;
252        }
253      } else {
254        displayUsage(cmd);
255        return exitCode;
256      }
257    } else {
258      displayUsage(cmd);
259      return exitCode;
260    }
261
262    // initialize cluster
263    cluster = createCluster();
264        
265    // Submit the request
266    try {
267      if (submitJobFile != null) {
268        Job job = Job.getInstance(new JobConf(submitJobFile));
269        job.submit();
270        System.out.println("Created job " + job.getJobID());
271        exitCode = 0;
272      } else if (getStatus) {
273        Job job = getJob(JobID.forName(jobid));
274        if (job == null) {
275          System.out.println("Could not find job " + jobid);
276        } else {
277          Counters counters = job.getCounters();
278          System.out.println();
279          System.out.println(job);
280          if (counters != null) {
281            System.out.println(counters);
282          } else {
283            System.out.println("Counters not available. Job is retired.");
284          }
285          exitCode = 0;
286        }
287      } else if (getCounter) {
288        Job job = getJob(JobID.forName(jobid));
289        if (job == null) {
290          System.out.println("Could not find job " + jobid);
291        } else {
292          Counters counters = job.getCounters();
293          if (counters == null) {
294            System.out.println("Counters not available for retired job " + 
295            jobid);
296            exitCode = -1;
297          } else {
298            System.out.println(getCounter(counters,
299              counterGroupName, counterName));
300            exitCode = 0;
301          }
302        }
303      } else if (killJob) {
304        Job job = getJob(JobID.forName(jobid));
305        if (job == null) {
306          System.out.println("Could not find job " + jobid);
307        } else {
308          JobStatus jobStatus = job.getStatus();
309          if (jobStatus.getState() == JobStatus.State.FAILED) {
310            System.out.println("Could not mark the job " + jobid
311                + " as killed, as it has already failed.");
312            exitCode = -1;
313          } else if (jobStatus.getState() == JobStatus.State.KILLED) {
314            System.out
315                .println("The job " + jobid + " has already been killed.");
316            exitCode = -1;
317          } else if (jobStatus.getState() == JobStatus.State.SUCCEEDED) {
318            System.out.println("Could not kill the job " + jobid
319                + ", as it has already succeeded.");
320            exitCode = -1;
321          } else {
322            job.killJob();
323            System.out.println("Killed job " + jobid);
324            exitCode = 0;
325          }
326        }
327      } else if (setJobPriority) {
328        Job job = getJob(JobID.forName(jobid));
329        if (job == null) {
330          System.out.println("Could not find job " + jobid);
331        } else {
332          if (jp != null) {
333            job.setPriority(jp);
334          } else {
335            job.setPriorityAsInteger(jpvalue);
336          }
337          System.out.println("Changed job priority.");
338          exitCode = 0;
339        } 
340      } else if (viewHistory) {
341        viewHistory(historyFile, viewAllHistory);
342        exitCode = 0;
343      } else if (listEvents) {
344        listEvents(getJob(JobID.forName(jobid)), fromEvent, nEvents);
345        exitCode = 0;
346      } else if (listJobs) {
347        listJobs(cluster);
348        exitCode = 0;
349      } else if (listAllJobs) {
350        listAllJobs(cluster);
351        exitCode = 0;
352      } else if (listActiveTrackers) {
353        listActiveTrackers(cluster);
354        exitCode = 0;
355      } else if (listBlacklistedTrackers) {
356        listBlacklistedTrackers(cluster);
357        exitCode = 0;
358      } else if (displayTasks) {
359        displayTasks(getJob(JobID.forName(jobid)), taskType, taskState);
360        exitCode = 0;
361      } else if(killTask) {
362        TaskAttemptID taskID = TaskAttemptID.forName(taskid);
363        Job job = getJob(taskID.getJobID());
364        if (job == null) {
365          System.out.println("Could not find job " + jobid);
366        } else if (job.killTask(taskID, false)) {
367          System.out.println("Killed task " + taskid);
368          exitCode = 0;
369        } else {
370          System.out.println("Could not kill task " + taskid);
371          exitCode = -1;
372        }
373      } else if(failTask) {
374        TaskAttemptID taskID = TaskAttemptID.forName(taskid);
375        Job job = getJob(taskID.getJobID());
376        if (job == null) {
377            System.out.println("Could not find job " + jobid);
378        } else if(job.killTask(taskID, true)) {
379          System.out.println("Killed task " + taskID + " by failing it");
380          exitCode = 0;
381        } else {
382          System.out.println("Could not fail task " + taskid);
383          exitCode = -1;
384        }
385      } else if (logs) {
386        try {
387        JobID jobID = JobID.forName(jobid);
388        TaskAttemptID taskAttemptID = TaskAttemptID.forName(taskid);
389        LogParams logParams = cluster.getLogParams(jobID, taskAttemptID);
390        LogCLIHelpers logDumper = new LogCLIHelpers();
391        logDumper.setConf(getConf());
392        exitCode = logDumper.dumpAContainersLogs(logParams.getApplicationId(),
393            logParams.getContainerId(), logParams.getNodeId(),
394            logParams.getOwner());
395        } catch (IOException e) {
396          if (e instanceof RemoteException) {
397            throw e;
398          } 
399          System.out.println(e.getMessage());
400        }
401      }
402    } catch (RemoteException re) {
403      IOException unwrappedException = re.unwrapRemoteException();
404      if (unwrappedException instanceof AccessControlException) {
405        System.out.println(unwrappedException.getMessage());
406      } else {
407        throw re;
408      }
409    } finally {
410      cluster.close();
411    }
412    return exitCode;
413  }
414
415  Cluster createCluster() throws IOException {
416    return new Cluster(getConf());
417  }
418
419  private String getJobPriorityNames() {
420    StringBuffer sb = new StringBuffer();
421    for (JobPriority p : JobPriority.values()) {
422      // UNDEFINED_PRIORITY need not to be displayed in usage
423      if (JobPriority.UNDEFINED_PRIORITY == p) {
424        continue;
425      }
426      sb.append(p.name()).append(" ");
427    }
428    return sb.substring(0, sb.length()-1);
429  }
430
431  private String getTaskTypes() {
432    return StringUtils.join(taskTypes, " ");
433  }
434
435  /**
436   * Display usage of the command-line tool and terminate execution.
437   */
438  private void displayUsage(String cmd) {
439    String prefix = "Usage: job ";
440    String jobPriorityValues = getJobPriorityNames();
441    String taskStates = "pending, running, completed, failed, killed";
442
443    if ("-submit".equals(cmd)) {
444      System.err.println(prefix + "[" + cmd + " <job-file>]");
445    } else if ("-status".equals(cmd) || "-kill".equals(cmd)) {
446      System.err.println(prefix + "[" + cmd + " <job-id>]");
447    } else if ("-counter".equals(cmd)) {
448      System.err.println(prefix + "[" + cmd + 
449        " <job-id> <group-name> <counter-name>]");
450    } else if ("-events".equals(cmd)) {
451      System.err.println(prefix + "[" + cmd + 
452        " <job-id> <from-event-#> <#-of-events>]. Event #s start from 1.");
453    } else if ("-history".equals(cmd)) {
454      System.err.println(prefix + "[" + cmd + " <jobHistoryFile>]");
455    } else if ("-list".equals(cmd)) {
456      System.err.println(prefix + "[" + cmd + " [all]]");
457    } else if ("-kill-task".equals(cmd) || "-fail-task".equals(cmd)) {
458      System.err.println(prefix + "[" + cmd + " <task-attempt-id>]");
459    } else if ("-set-priority".equals(cmd)) {
460      System.err.println(prefix + "[" + cmd + " <job-id> <priority>]. " +
461          "Valid values for priorities are: " 
462          + jobPriorityValues
463          + ". In addition to this, integers also can be used.");
464    } else if ("-list-active-trackers".equals(cmd)) {
465      System.err.println(prefix + "[" + cmd + "]");
466    } else if ("-list-blacklisted-trackers".equals(cmd)) {
467      System.err.println(prefix + "[" + cmd + "]");
468    } else if ("-list-attempt-ids".equals(cmd)) {
469      System.err.println(prefix + "[" + cmd + 
470          " <job-id> <task-type> <task-state>]. " +
471          "Valid values for <task-type> are " + getTaskTypes() + ". " +
472          "Valid values for <task-state> are " + taskStates);
473    } else if ("-logs".equals(cmd)) {
474      System.err.println(prefix + "[" + cmd +
475          " <job-id> <task-attempt-id>]. " +
476          " <task-attempt-id> is optional to get task attempt logs.");      
477    } else {
478      System.err.printf(prefix + "<command> <args>%n");
479      System.err.printf("\t[-submit <job-file>]%n");
480      System.err.printf("\t[-status <job-id>]%n");
481      System.err.printf("\t[-counter <job-id> <group-name> <counter-name>]%n");
482      System.err.printf("\t[-kill <job-id>]%n");
483      System.err.printf("\t[-set-priority <job-id> <priority>]. " +
484          "Valid values for priorities are: " + jobPriorityValues +
485          ". In addition to this, integers also can be used." + "%n");
486      System.err.printf("\t[-events <job-id> <from-event-#> <#-of-events>]%n");
487      System.err.printf("\t[-history <jobHistoryFile>]%n");
488      System.err.printf("\t[-list [all]]%n");
489      System.err.printf("\t[-list-active-trackers]%n");
490      System.err.printf("\t[-list-blacklisted-trackers]%n");
491      System.err.println("\t[-list-attempt-ids <job-id> <task-type> " +
492        "<task-state>]. " +
493        "Valid values for <task-type> are " + getTaskTypes() + ". " +
494        "Valid values for <task-state> are " + taskStates);
495      System.err.printf("\t[-kill-task <task-attempt-id>]%n");
496      System.err.printf("\t[-fail-task <task-attempt-id>]%n");
497      System.err.printf("\t[-logs <job-id> <task-attempt-id>]%n%n");
498      ToolRunner.printGenericCommandUsage(System.out);
499    }
500  }
501    
502  private void viewHistory(String historyFile, boolean all) 
503    throws IOException {
504    HistoryViewer historyViewer = new HistoryViewer(historyFile,
505                                        getConf(), all);
506    historyViewer.print();
507  }
508
509  protected long getCounter(Counters counters, String counterGroupName,
510      String counterName) throws IOException {
511    return counters.findCounter(counterGroupName, counterName).getValue();
512  }
513  
514  /**
515   * List the events for the given job
516   * @param job the job to list
517   * @param fromEventId event id for the job's events to list from
518   * @param numEvents number of events we want to list
519   * @throws IOException
520   */
521  private void listEvents(Job job, int fromEventId, int numEvents)
522      throws IOException, InterruptedException {
523    TaskCompletionEvent[] events = job.
524      getTaskCompletionEvents(fromEventId, numEvents);
525    System.out.println("Task completion events for " + job.getJobID());
526    System.out.println("Number of events (from " + fromEventId + ") are: " 
527      + events.length);
528    for(TaskCompletionEvent event: events) {
529      System.out.println(event.getStatus() + " " + 
530        event.getTaskAttemptId() + " " + 
531        getTaskLogURL(event.getTaskAttemptId(), event.getTaskTrackerHttp()));
532    }
533  }
534
535  protected static String getTaskLogURL(TaskAttemptID taskId, String baseUrl) {
536    return (baseUrl + "/tasklog?plaintext=true&attemptid=" + taskId); 
537  }
538
539  @VisibleForTesting
540  Job getJob(JobID jobid) throws IOException, InterruptedException {
541
542    int maxRetry = getConf().getInt(MRJobConfig.MR_CLIENT_JOB_MAX_RETRIES,
543        MRJobConfig.DEFAULT_MR_CLIENT_JOB_MAX_RETRIES);
544    long retryInterval = getConf()
545        .getLong(MRJobConfig.MR_CLIENT_JOB_RETRY_INTERVAL,
546            MRJobConfig.DEFAULT_MR_CLIENT_JOB_RETRY_INTERVAL);
547    Job job = cluster.getJob(jobid);
548
549    for (int i = 0; i < maxRetry; ++i) {
550      if (job != null) {
551        return job;
552      }
553      LOG.info("Could not obtain job info after " + String.valueOf(i + 1)
554          + " attempt(s). Sleeping for " + String.valueOf(retryInterval / 1000)
555          + " seconds and retrying.");
556      Thread.sleep(retryInterval);
557      job = cluster.getJob(jobid);
558    }
559    return job;
560  }
561  
562
563  /**
564   * Dump a list of currently running jobs
565   * @throws IOException
566   */
567  private void listJobs(Cluster cluster) 
568      throws IOException, InterruptedException {
569    List<JobStatus> runningJobs = new ArrayList<JobStatus>();
570    for (JobStatus job : cluster.getAllJobStatuses()) {
571      if (!job.isJobComplete()) {
572        runningJobs.add(job);
573      }
574    }
575    displayJobList(runningJobs.toArray(new JobStatus[0]));
576  }
577    
578  /**
579   * Dump a list of all jobs submitted.
580   * @throws IOException
581   */
582  private void listAllJobs(Cluster cluster) 
583      throws IOException, InterruptedException {
584    displayJobList(cluster.getAllJobStatuses());
585  }
586  
587  /**
588   * Display the list of active trackers
589   */
590  private void listActiveTrackers(Cluster cluster) 
591      throws IOException, InterruptedException {
592    TaskTrackerInfo[] trackers = cluster.getActiveTaskTrackers();
593    for (TaskTrackerInfo tracker : trackers) {
594      System.out.println(tracker.getTaskTrackerName());
595    }
596  }
597
598  /**
599   * Display the list of blacklisted trackers
600   */
601  private void listBlacklistedTrackers(Cluster cluster) 
602      throws IOException, InterruptedException {
603    TaskTrackerInfo[] trackers = cluster.getBlackListedTaskTrackers();
604    if (trackers.length > 0) {
605      System.out.println("BlackListedNode \t Reason");
606    }
607    for (TaskTrackerInfo tracker : trackers) {
608      System.out.println(tracker.getTaskTrackerName() + "\t" + 
609        tracker.getReasonForBlacklist());
610    }
611  }
612
613  private void printTaskAttempts(TaskReport report) {
614    if (report.getCurrentStatus() == TIPStatus.COMPLETE) {
615      System.out.println(report.getSuccessfulTaskAttemptId());
616    } else if (report.getCurrentStatus() == TIPStatus.RUNNING) {
617      for (TaskAttemptID t : 
618        report.getRunningTaskAttemptIds()) {
619        System.out.println(t);
620      }
621    }
622  }
623
624  /**
625   * Display the information about a job's tasks, of a particular type and
626   * in a particular state
627   * 
628   * @param job the job
629   * @param type the type of the task (map/reduce/setup/cleanup)
630   * @param state the state of the task 
631   * (pending/running/completed/failed/killed)
632   */
633  protected void displayTasks(Job job, String type, String state) 
634  throws IOException, InterruptedException {
635    TaskReport[] reports = job.getTaskReports(TaskType.valueOf(
636        org.apache.hadoop.util.StringUtils.toUpperCase(type)));
637    for (TaskReport report : reports) {
638      TIPStatus status = report.getCurrentStatus();
639      if ((state.equalsIgnoreCase("pending") && status ==TIPStatus.PENDING) ||
640          (state.equalsIgnoreCase("running") && status ==TIPStatus.RUNNING) ||
641          (state.equalsIgnoreCase("completed") && status == TIPStatus.COMPLETE) ||
642          (state.equalsIgnoreCase("failed") && status == TIPStatus.FAILED) ||
643          (state.equalsIgnoreCase("killed") && status == TIPStatus.KILLED)) {
644        printTaskAttempts(report);
645      }
646    }
647  }
648
649  public void displayJobList(JobStatus[] jobs) 
650      throws IOException, InterruptedException {
651    displayJobList(jobs, new PrintWriter(new OutputStreamWriter(System.out,
652        Charsets.UTF_8)));
653  }
654
655  @Private
656  public static String headerPattern = "%23s\t%10s\t%14s\t%12s\t%12s\t%10s\t%15s\t%15s\t%8s\t%8s\t%10s\t%10s\n";
657  @Private
658  public static String dataPattern   = "%23s\t%10s\t%14d\t%12s\t%12s\t%10s\t%15s\t%15s\t%8s\t%8s\t%10s\t%10s\n";
659  private static String memPattern   = "%dM";
660  private static String UNAVAILABLE  = "N/A";
661
662  @Private
663  public void displayJobList(JobStatus[] jobs, PrintWriter writer) {
664    writer.println("Total jobs:" + jobs.length);
665    writer.printf(headerPattern, "JobId", "State", "StartTime", "UserName",
666      "Queue", "Priority", "UsedContainers",
667      "RsvdContainers", "UsedMem", "RsvdMem", "NeededMem", "AM info");
668    for (JobStatus job : jobs) {
669      int numUsedSlots = job.getNumUsedSlots();
670      int numReservedSlots = job.getNumReservedSlots();
671
672      long usedMem = job.getUsedMem();
673      long rsvdMem = job.getReservedMem();
674      long neededMem = job.getNeededMem();
675      writer.printf(dataPattern,
676          job.getJobID().toString(), job.getState(), job.getStartTime(),
677          job.getUsername(), job.getQueue(), 
678          job.getPriority().name(),
679          numUsedSlots < 0 ? UNAVAILABLE : numUsedSlots,
680          numReservedSlots < 0 ? UNAVAILABLE : numReservedSlots,
681          usedMem < 0 ? UNAVAILABLE : String.format(memPattern, usedMem),
682          rsvdMem < 0 ? UNAVAILABLE : String.format(memPattern, rsvdMem),
683          neededMem < 0 ? UNAVAILABLE : String.format(memPattern, neededMem),
684          job.getSchedulingInfo());
685    }
686    writer.flush();
687  }
688  
689  public static void main(String[] argv) throws Exception {
690    int res = ToolRunner.run(new CLI(), argv);
691    ExitUtil.terminate(res);
692  }
693}