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
019package org.apache.hadoop.mapred;
020
021import java.io.DataInput;
022import java.io.DataOutput;
023import java.io.IOException;
024import java.util.ArrayList;
025import java.util.Collection;
026import java.util.Collections;
027
028import org.apache.hadoop.classification.InterfaceAudience;
029import org.apache.hadoop.classification.InterfaceStability;
030import org.apache.hadoop.io.Text;
031import org.apache.hadoop.io.Writable;
032import org.apache.hadoop.io.WritableUtils;
033import org.apache.hadoop.mapreduce.Cluster.JobTrackerStatus;
034import org.apache.hadoop.util.StringInterner;
035
036/**
037 * Status information on the current state of the Map-Reduce cluster.
038 * 
039 * <p><code>ClusterStatus</code> provides clients with information such as:
040 * <ol>
041 *   <li>
042 *   Size of the cluster. 
043 *   </li>
044 *   <li>
045 *   Name of the trackers. 
046 *   </li>
047 *   <li>
048 *   Task capacity of the cluster. 
049 *   </li>
050 *   <li>
051 *   The number of currently running map and reduce tasks.
052 *   </li>
053 *   <li>
054 *   State of the <code>JobTracker</code>.
055 *   </li>
056 *   <li>
057 *   Details regarding black listed trackers.
058 *   </li>
059 * </ol>
060 * 
061 * <p>Clients can query for the latest <code>ClusterStatus</code>, via 
062 * {@link JobClient#getClusterStatus()}.</p>
063 * 
064 * @see JobClient
065 */
066@InterfaceAudience.Public
067@InterfaceStability.Stable
068public class ClusterStatus implements Writable {
069  /**
070   * Class which encapsulates information about a blacklisted tasktracker.
071   *  
072   * The information includes the tasktracker's name and reasons for
073   * getting blacklisted. The toString method of the class will print
074   * the information in a whitespace separated fashion to enable parsing.
075   */
076  public static class BlackListInfo implements Writable {
077
078    private String trackerName;
079
080    private String reasonForBlackListing;
081    
082    private String blackListReport;
083    
084    BlackListInfo() {
085    }
086    
087
088    /**
089     * Gets the blacklisted tasktracker's name.
090     * 
091     * @return tracker's name.
092     */
093    public String getTrackerName() {
094      return trackerName;
095    }
096
097    /**
098     * Gets the reason for which the tasktracker was blacklisted.
099     * 
100     * @return reason which tracker was blacklisted
101     */
102    public String getReasonForBlackListing() {
103      return reasonForBlackListing;
104    }
105
106    /**
107     * Sets the blacklisted tasktracker's name.
108     * 
109     * @param trackerName of the tracker.
110     */
111    void setTrackerName(String trackerName) {
112      this.trackerName = trackerName;
113    }
114
115    /**
116     * Sets the reason for which the tasktracker was blacklisted.
117     * 
118     * @param reasonForBlackListing
119     */
120    void setReasonForBlackListing(String reasonForBlackListing) {
121      this.reasonForBlackListing = reasonForBlackListing;
122    }
123
124    /**
125     * Gets a descriptive report about why the tasktracker was blacklisted.
126     * 
127     * @return report describing why the tasktracker was blacklisted.
128     */
129    public String getBlackListReport() {
130      return blackListReport;
131    }
132
133    /**
134     * Sets a descriptive report about why the tasktracker was blacklisted.
135     * @param blackListReport report describing why the tasktracker 
136     *                        was blacklisted.
137     */
138    void setBlackListReport(String blackListReport) {
139      this.blackListReport = blackListReport;
140    }
141
142    @Override
143    public void readFields(DataInput in) throws IOException {
144      trackerName = StringInterner.weakIntern(Text.readString(in));
145      reasonForBlackListing = StringInterner.weakIntern(Text.readString(in));
146      blackListReport = StringInterner.weakIntern(Text.readString(in));
147    }
148
149    @Override
150    public void write(DataOutput out) throws IOException {
151      Text.writeString(out, trackerName);
152      Text.writeString(out, reasonForBlackListing);
153      Text.writeString(out, blackListReport);
154    }
155
156    @Override
157    /**
158     * Print information related to the blacklisted tasktracker in a
159     * whitespace separated fashion.
160     * 
161     * The method changes any newlines in the report describing why
162     * the tasktracker was blacklisted to a ':' for enabling better
163     * parsing.
164     */
165    public String toString() {
166      StringBuilder sb = new StringBuilder();
167      sb.append(trackerName);
168      sb.append("\t");
169      sb.append(reasonForBlackListing);
170      sb.append("\t");
171      sb.append(blackListReport.replace("\n", ":"));
172      return sb.toString();
173    }
174    
175  }
176  
177  public static final long UNINITIALIZED_MEMORY_VALUE = -1;
178  
179  private int numActiveTrackers;
180  private Collection<String> activeTrackers = new ArrayList<String>();
181  private int numBlacklistedTrackers;
182  private int numExcludedNodes;
183  private long ttExpiryInterval;
184  private int map_tasks;
185  private int reduce_tasks;
186  private int max_map_tasks;
187  private int max_reduce_tasks;
188  private JobTrackerStatus status;
189  private Collection<BlackListInfo> blacklistedTrackersInfo =
190    new ArrayList<BlackListInfo>();
191  private int grayListedTrackers;
192
193  ClusterStatus() {}
194  
195  /**
196   * Construct a new cluster status.
197   * 
198   * @param trackers no. of tasktrackers in the cluster
199   * @param blacklists no of blacklisted task trackers in the cluster
200   * @param ttExpiryInterval the tasktracker expiry interval
201   * @param maps no. of currently running map-tasks in the cluster
202   * @param reduces no. of currently running reduce-tasks in the cluster
203   * @param maxMaps the maximum no. of map tasks in the cluster
204   * @param maxReduces the maximum no. of reduce tasks in the cluster
205   * @param status the {@link JobTrackerStatus} of the <code>JobTracker</code>
206   */
207  ClusterStatus(int trackers, int blacklists, long ttExpiryInterval, 
208                int maps, int reduces,
209                int maxMaps, int maxReduces, JobTrackerStatus status) {
210    this(trackers, blacklists, ttExpiryInterval, maps, reduces, maxMaps, 
211         maxReduces, status, 0);
212  }
213
214  /**
215   * Construct a new cluster status.
216   * 
217   * @param trackers no. of tasktrackers in the cluster
218   * @param blacklists no of blacklisted task trackers in the cluster
219   * @param ttExpiryInterval the tasktracker expiry interval
220   * @param maps no. of currently running map-tasks in the cluster
221   * @param reduces no. of currently running reduce-tasks in the cluster
222   * @param maxMaps the maximum no. of map tasks in the cluster
223   * @param maxReduces the maximum no. of reduce tasks in the cluster
224   * @param status the {@link JobTrackerStatus} of the <code>JobTracker</code>
225   * @param numDecommissionedNodes number of decommission trackers
226   */
227  ClusterStatus(int trackers, int blacklists, long ttExpiryInterval, int maps,
228      int reduces, int maxMaps, int maxReduces, JobTrackerStatus status,
229      int numDecommissionedNodes) {
230    this(trackers, blacklists, ttExpiryInterval, maps, reduces, maxMaps,
231      maxReduces, status, numDecommissionedNodes, 0);
232  }
233
234  /**
235   * Construct a new cluster status.
236   * 
237   * @param trackers no. of tasktrackers in the cluster
238   * @param blacklists no of blacklisted task trackers in the cluster
239   * @param ttExpiryInterval the tasktracker expiry interval
240   * @param maps no. of currently running map-tasks in the cluster
241   * @param reduces no. of currently running reduce-tasks in the cluster
242   * @param maxMaps the maximum no. of map tasks in the cluster
243   * @param maxReduces the maximum no. of reduce tasks in the cluster
244   * @param status the {@link JobTrackerStatus} of the <code>JobTracker</code>
245   * @param numDecommissionedNodes number of decommission trackers
246   * @param numGrayListedTrackers number of graylisted trackers
247   */
248  ClusterStatus(int trackers, int blacklists, long ttExpiryInterval, int maps,
249      int reduces, int maxMaps, int maxReduces, JobTrackerStatus status,
250      int numDecommissionedNodes, int numGrayListedTrackers) {
251    numActiveTrackers = trackers;
252    numBlacklistedTrackers = blacklists;
253    this.numExcludedNodes = numDecommissionedNodes;
254    this.ttExpiryInterval = ttExpiryInterval;
255    map_tasks = maps;
256    reduce_tasks = reduces;
257    max_map_tasks = maxMaps;
258    max_reduce_tasks = maxReduces;
259    this.status = status;
260    this.grayListedTrackers = numGrayListedTrackers;
261  }
262
263  /**
264   * Construct a new cluster status.
265   * 
266   * @param activeTrackers active tasktrackers in the cluster
267   * @param blacklistedTrackers blacklisted tasktrackers in the cluster
268   * @param ttExpiryInterval the tasktracker expiry interval
269   * @param maps no. of currently running map-tasks in the cluster
270   * @param reduces no. of currently running reduce-tasks in the cluster
271   * @param maxMaps the maximum no. of map tasks in the cluster
272   * @param maxReduces the maximum no. of reduce tasks in the cluster
273   * @param status the {@link JobTrackerStatus} of the <code>JobTracker</code>
274   */
275  ClusterStatus(Collection<String> activeTrackers, 
276      Collection<BlackListInfo> blacklistedTrackers,
277      long ttExpiryInterval,
278      int maps, int reduces, int maxMaps, int maxReduces, 
279      JobTrackerStatus status) {
280    this(activeTrackers, blacklistedTrackers, ttExpiryInterval, maps, reduces, 
281         maxMaps, maxReduces, status, 0);
282  }
283
284
285  /**
286   * Construct a new cluster status.
287   * 
288   * @param activeTrackers active tasktrackers in the cluster
289   * @param blackListedTrackerInfo blacklisted tasktrackers information 
290   * in the cluster
291   * @param ttExpiryInterval the tasktracker expiry interval
292   * @param maps no. of currently running map-tasks in the cluster
293   * @param reduces no. of currently running reduce-tasks in the cluster
294   * @param maxMaps the maximum no. of map tasks in the cluster
295   * @param maxReduces the maximum no. of reduce tasks in the cluster
296   * @param status the {@link JobTrackerStatus} of the <code>JobTracker</code>
297   * @param numDecommissionNodes number of decommission trackers
298   */
299  
300  ClusterStatus(Collection<String> activeTrackers,
301      Collection<BlackListInfo> blackListedTrackerInfo, long ttExpiryInterval,
302      int maps, int reduces, int maxMaps, int maxReduces,
303      JobTrackerStatus status, int numDecommissionNodes) {
304    this(activeTrackers.size(), blackListedTrackerInfo.size(),
305        ttExpiryInterval, maps, reduces, maxMaps, maxReduces, status,
306        numDecommissionNodes);
307    this.activeTrackers = activeTrackers;
308    this.blacklistedTrackersInfo = blackListedTrackerInfo;
309  }
310
311  /**
312   * Get the number of task trackers in the cluster.
313   * 
314   * @return the number of task trackers in the cluster.
315   */
316  public int getTaskTrackers() {
317    return numActiveTrackers;
318  }
319  
320  /**
321   * Get the names of task trackers in the cluster.
322   * 
323   * @return the active task trackers in the cluster.
324   */
325  public Collection<String> getActiveTrackerNames() {
326    return activeTrackers;
327  }
328
329  /**
330   * Get the names of task trackers in the cluster.
331   * 
332   * @return the blacklisted task trackers in the cluster.
333   */
334  public Collection<String> getBlacklistedTrackerNames() {
335    ArrayList<String> blacklistedTrackers = new ArrayList<String>();
336    for(BlackListInfo bi : blacklistedTrackersInfo) {
337      blacklistedTrackers.add(bi.getTrackerName());
338    }
339    return blacklistedTrackers;
340  }
341
342  /**
343   * Get the names of graylisted task trackers in the cluster.
344   *
345   * The gray list of trackers is no longer available on M/R 2.x. The function
346   * is kept to be compatible with M/R 1.x applications.
347   *
348   * @return an empty graylisted task trackers in the cluster.
349   */
350  @Deprecated
351  public Collection<String> getGraylistedTrackerNames() {
352    return Collections.emptySet();
353  }
354
355  /**
356   * Get the number of graylisted task trackers in the cluster.
357   *
358   * The gray list of trackers is no longer available on M/R 2.x. The function
359   * is kept to be compatible with M/R 1.x applications.
360   *
361   * @return 0 graylisted task trackers in the cluster.
362   */
363  @Deprecated
364  public int getGraylistedTrackers() {
365    return grayListedTrackers;
366  }
367
368  /**
369   * Get the number of blacklisted task trackers in the cluster.
370   * 
371   * @return the number of blacklisted task trackers in the cluster.
372   */
373  public int getBlacklistedTrackers() {
374    return numBlacklistedTrackers;
375  }
376  
377  /**
378   * Get the number of excluded hosts in the cluster.
379   * @return the number of excluded hosts in the cluster.
380   */
381  public int getNumExcludedNodes() {
382    return numExcludedNodes;
383  }
384  
385  /**
386   * Get the tasktracker expiry interval for the cluster
387   * @return the expiry interval in msec
388   */
389  public long getTTExpiryInterval() {
390    return ttExpiryInterval;
391  }
392  
393  /**
394   * Get the number of currently running map tasks in the cluster.
395   * 
396   * @return the number of currently running map tasks in the cluster.
397   */
398  public int getMapTasks() {
399    return map_tasks;
400  }
401  
402  /**
403   * Get the number of currently running reduce tasks in the cluster.
404   * 
405   * @return the number of currently running reduce tasks in the cluster.
406   */
407  public int getReduceTasks() {
408    return reduce_tasks;
409  }
410  
411  /**
412   * Get the maximum capacity for running map tasks in the cluster.
413   * 
414   * @return the maximum capacity for running map tasks in the cluster.
415   */
416  public int getMaxMapTasks() {
417    return max_map_tasks;
418  }
419
420  /**
421   * Get the maximum capacity for running reduce tasks in the cluster.
422   * 
423   * @return the maximum capacity for running reduce tasks in the cluster.
424   */
425  public int getMaxReduceTasks() {
426    return max_reduce_tasks;
427  }
428  
429  /**
430   * Get the JobTracker's status.
431   * 
432   * @return {@link JobTrackerStatus} of the JobTracker
433   */
434  public JobTrackerStatus getJobTrackerStatus() {
435    return status;
436  }
437  
438  /**
439   * Returns UNINITIALIZED_MEMORY_VALUE (-1)
440   */
441  @Deprecated
442  public long getMaxMemory() {
443    return UNINITIALIZED_MEMORY_VALUE;
444  }
445  
446  /**
447   * Returns UNINITIALIZED_MEMORY_VALUE (-1)
448   */
449  @Deprecated
450  public long getUsedMemory() {
451    return UNINITIALIZED_MEMORY_VALUE;
452  }
453
454  /**
455   * Gets the list of blacklisted trackers along with reasons for blacklisting.
456   * 
457   * @return the collection of {@link BlackListInfo} objects. 
458   * 
459   */
460  public Collection<BlackListInfo> getBlackListedTrackersInfo() {
461    return blacklistedTrackersInfo;
462  }
463
464  /**
465   * Get the current state of the <code>JobTracker</code>,
466   * as {@link JobTracker.State}
467   *
468   * {@link JobTracker.State} should no longer be used on M/R 2.x. The function
469   * is kept to be compatible with M/R 1.x applications.
470   *
471   * @return the invalid state of the <code>JobTracker</code>.
472   */
473  @Deprecated
474  public JobTracker.State getJobTrackerState() {
475    return JobTracker.State.RUNNING;
476  }
477
478  public void write(DataOutput out) throws IOException {
479    if (activeTrackers.size() == 0) {
480      out.writeInt(numActiveTrackers);
481      out.writeInt(0);
482    } else {
483      out.writeInt(activeTrackers.size());
484      out.writeInt(activeTrackers.size());
485      for (String tracker : activeTrackers) {
486        Text.writeString(out, tracker);
487      }
488    }
489    if (blacklistedTrackersInfo.size() == 0) {
490      out.writeInt(numBlacklistedTrackers);
491      out.writeInt(blacklistedTrackersInfo.size());
492    } else {
493      out.writeInt(blacklistedTrackersInfo.size());
494      out.writeInt(blacklistedTrackersInfo.size());
495      for (BlackListInfo tracker : blacklistedTrackersInfo) {
496        tracker.write(out);
497      }
498    }
499    out.writeInt(numExcludedNodes);
500    out.writeLong(ttExpiryInterval);
501    out.writeInt(map_tasks);
502    out.writeInt(reduce_tasks);
503    out.writeInt(max_map_tasks);
504    out.writeInt(max_reduce_tasks);
505    WritableUtils.writeEnum(out, status);
506    out.writeInt(grayListedTrackers);
507  }
508
509  public void readFields(DataInput in) throws IOException {
510    numActiveTrackers = in.readInt();
511    int numTrackerNames = in.readInt();
512    if (numTrackerNames > 0) {
513      for (int i = 0; i < numTrackerNames; i++) {
514        String name = StringInterner.weakIntern(Text.readString(in));
515        activeTrackers.add(name);
516      }
517    }
518    numBlacklistedTrackers = in.readInt();
519    int blackListTrackerInfoSize = in.readInt();
520    if(blackListTrackerInfoSize > 0) {
521      for (int i = 0; i < blackListTrackerInfoSize; i++) {
522        BlackListInfo info = new BlackListInfo();
523        info.readFields(in);
524        blacklistedTrackersInfo.add(info);
525      }
526    }
527    numExcludedNodes = in.readInt();
528    ttExpiryInterval = in.readLong();
529    map_tasks = in.readInt();
530    reduce_tasks = in.readInt();
531    max_map_tasks = in.readInt();
532    max_reduce_tasks = in.readInt();
533    status = WritableUtils.readEnum(in, JobTrackerStatus.class);
534    grayListedTrackers = in.readInt();
535  }
536}