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.mapreduce;
020
021import java.io.DataInput;
022import java.io.DataOutput;
023import java.io.IOException;
024import java.text.NumberFormat;
025import java.util.EnumMap;
026import java.util.HashMap;
027import java.util.Map;
028
029import org.apache.hadoop.classification.InterfaceAudience;
030import org.apache.hadoop.classification.InterfaceStability;
031import org.apache.hadoop.io.WritableUtils;
032
033
034/**
035 * TaskID represents the immutable and unique identifier for 
036 * a Map or Reduce Task. Each TaskID encompasses multiple attempts made to
037 * execute the Map or Reduce Task, each of which are uniquely indentified by
038 * their TaskAttemptID.
039 * 
040 * TaskID consists of 3 parts. First part is the {@link JobID}, that this 
041 * TaskInProgress belongs to. Second part of the TaskID is either 'm' or 'r' 
042 * representing whether the task is a map task or a reduce task. 
043 * And the third part is the task number. <br> 
044 * An example TaskID is : 
045 * <code>task_200707121733_0003_m_000005</code> , which represents the
046 * fifth map task in the third job running at the jobtracker 
047 * started at <code>200707121733</code>. 
048 * <p>
049 * Applications should never construct or parse TaskID strings
050 * , but rather use appropriate constructors or {@link #forName(String)} 
051 * method. 
052 * 
053 * @see JobID
054 * @see TaskAttemptID
055 */
056@InterfaceAudience.Public
057@InterfaceStability.Stable
058public class TaskID extends org.apache.hadoop.mapred.ID {
059  protected static final String TASK = "task";
060  protected static final NumberFormat idFormat = NumberFormat.getInstance();
061  static {
062    idFormat.setGroupingUsed(false);
063    idFormat.setMinimumIntegerDigits(6);
064  }
065  
066  private JobID jobId;
067  private TaskType type;
068  
069  /**
070   * Constructs a TaskID object from given {@link JobID}.  
071   * @param jobId JobID that this tip belongs to 
072   * @param type the {@link TaskType} of the task 
073   * @param id the tip number
074   */
075  public TaskID(JobID jobId, TaskType type, int id) {
076    super(id);
077    if(jobId == null) {
078      throw new IllegalArgumentException("jobId cannot be null");
079    }
080    this.jobId = jobId;
081    this.type = type;
082  }
083  
084  /**
085   * Constructs a TaskInProgressId object from given parts.
086   * @param jtIdentifier jobTracker identifier
087   * @param jobId job number 
088   * @param type the TaskType 
089   * @param id the tip number
090   */
091  public TaskID(String jtIdentifier, int jobId, TaskType type, int id) {
092    this(new JobID(jtIdentifier, jobId), type, id);
093  }
094  
095  public TaskID() { 
096    jobId = new JobID();
097  }
098  
099  /** Returns the {@link JobID} object that this tip belongs to */
100  public JobID getJobID() {
101    return jobId;
102  }
103  
104  /**Returns whether this TaskID is a map ID */
105  @Deprecated
106  public boolean isMap() {
107    return type == TaskType.MAP;
108  }
109    
110  /**
111   * Get the type of the task
112   */
113  public TaskType getTaskType() {
114    return type;
115  }
116  
117  @Override
118  public boolean equals(Object o) {
119    if (!super.equals(o))
120      return false;
121
122    TaskID that = (TaskID)o;
123    return this.type == that.type && this.jobId.equals(that.jobId);
124  }
125
126  /**Compare TaskInProgressIds by first jobIds, then by tip numbers. Reduces are 
127   * defined as greater then maps.*/
128  @Override
129  public int compareTo(ID o) {
130    TaskID that = (TaskID)o;
131    int jobComp = this.jobId.compareTo(that.jobId);
132    if(jobComp == 0) {
133      if(this.type == that.type) {
134        return this.id - that.id;
135      }
136      else {
137        return this.type.compareTo(that.type);
138      }
139    }
140    else return jobComp;
141  }
142  @Override
143  public String toString() { 
144    return appendTo(new StringBuilder(TASK)).toString();
145  }
146
147  /**
148   * Add the unique string to the given builder.
149   * @param builder the builder to append to
150   * @return the builder that was passed in
151   */
152  protected StringBuilder appendTo(StringBuilder builder) {
153    return jobId.appendTo(builder).
154                 append(SEPARATOR).
155                 append(CharTaskTypeMaps.getRepresentingCharacter(type)).
156                 append(SEPARATOR).
157                 append(idFormat.format(id));
158  }
159  
160  @Override
161  public int hashCode() {
162    return jobId.hashCode() * 524287 + id;
163  }
164  
165  @Override
166  public void readFields(DataInput in) throws IOException {
167    super.readFields(in);
168    jobId.readFields(in);
169    type = WritableUtils.readEnum(in, TaskType.class);
170  }
171
172  @Override
173  public void write(DataOutput out) throws IOException {
174    super.write(out);
175    jobId.write(out);
176    WritableUtils.writeEnum(out, type);
177  }
178  
179  /** Construct a TaskID object from given string 
180   * @return constructed TaskID object or null if the given String is null
181   * @throws IllegalArgumentException if the given string is malformed
182   */
183  public static TaskID forName(String str) 
184    throws IllegalArgumentException {
185    if(str == null)
186      return null;
187    String exceptionMsg = null;
188    try {
189      String[] parts = str.split("_");
190      if(parts.length == 5) {
191        if(parts[0].equals(TASK)) {
192          String type = parts[3];
193          TaskType t = CharTaskTypeMaps.getTaskType(type.charAt(0));
194          if(t != null) {
195          
196            return new org.apache.hadoop.mapred.TaskID(parts[1], 
197                                                     Integer.parseInt(parts[2]),
198                                                     t, 
199                                                     Integer.parseInt(parts[4]));
200          } else
201            exceptionMsg = "Bad TaskType identifier. TaskId string : " + str
202                + " is not properly formed.";
203        }
204      }
205    }catch (Exception ex) {//fall below
206    }
207    if (exceptionMsg == null) {
208      exceptionMsg = "TaskId string : " + str + " is not properly formed";
209    }
210    throw new IllegalArgumentException(exceptionMsg);
211  }
212  /**
213   * Gets the character representing the {@link TaskType}
214   * @param type the TaskType
215   * @return the character
216   */
217  public static char getRepresentingCharacter(TaskType type) {
218    return CharTaskTypeMaps.getRepresentingCharacter(type);
219  }
220  /**
221   * Gets the {@link TaskType} corresponding to the character
222   * @param c the character
223   * @return the TaskType
224   */
225  public static TaskType getTaskType(char c) {
226    return CharTaskTypeMaps.getTaskType(c);
227  }
228  
229  public static String getAllTaskTypes() {
230    return CharTaskTypeMaps.allTaskTypes;
231  }
232
233  /**
234   * Maintains the mapping from the character representation of a task type to 
235   * the enum class TaskType constants
236   */
237  static class CharTaskTypeMaps {
238    private static EnumMap<TaskType, Character> typeToCharMap = 
239      new EnumMap<TaskType,Character>(TaskType.class);
240    private static Map<Character, TaskType> charToTypeMap = 
241      new HashMap<Character, TaskType>();
242    static String allTaskTypes = "(m|r|s|c|t)";
243    static {
244      setupTaskTypeToCharMapping();
245      setupCharToTaskTypeMapping();
246    }
247    
248    private static void setupTaskTypeToCharMapping() {
249      typeToCharMap.put(TaskType.MAP, 'm');
250      typeToCharMap.put(TaskType.REDUCE, 'r');
251      typeToCharMap.put(TaskType.JOB_SETUP, 's');
252      typeToCharMap.put(TaskType.JOB_CLEANUP, 'c');
253      typeToCharMap.put(TaskType.TASK_CLEANUP, 't');
254    }
255
256    private static void setupCharToTaskTypeMapping() {
257      charToTypeMap.put('m', TaskType.MAP);
258      charToTypeMap.put('r', TaskType.REDUCE);
259      charToTypeMap.put('s', TaskType.JOB_SETUP);
260      charToTypeMap.put('c', TaskType.JOB_CLEANUP);
261      charToTypeMap.put('t', TaskType.TASK_CLEANUP);
262    }
263
264    static char getRepresentingCharacter(TaskType type) {
265      return typeToCharMap.get(type);
266    }
267    static TaskType getTaskType(char c) {
268      return charToTypeMap.get(c);
269    }
270  }
271
272}