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.hdfs.protocol;
019
020import static org.apache.hadoop.hdfs.DFSUtil.percent2String;
021
022import java.util.Date;
023
024import org.apache.hadoop.classification.InterfaceAudience;
025import org.apache.hadoop.classification.InterfaceStability;
026import org.apache.hadoop.hdfs.DFSConfigKeys;
027import org.apache.hadoop.hdfs.DFSUtil;
028import org.apache.hadoop.net.NetUtils;
029import org.apache.hadoop.net.NetworkTopology;
030import org.apache.hadoop.net.Node;
031import org.apache.hadoop.net.NodeBase;
032import org.apache.hadoop.util.StringUtils;
033import org.apache.hadoop.util.Time;
034
035/** 
036 * This class extends the primary identifier of a Datanode with ephemeral
037 * state, eg usage information, current administrative state, and the
038 * network location that is communicated to clients.
039 */
040@InterfaceAudience.Private
041@InterfaceStability.Evolving
042public class DatanodeInfo extends DatanodeID implements Node {
043  private long capacity;
044  private long dfsUsed;
045  private long remaining;
046  private long blockPoolUsed;
047  private long lastUpdate;
048  private int xceiverCount;
049  private String location = NetworkTopology.DEFAULT_RACK;
050  
051  // Datanode administrative states
052  public enum AdminStates {
053    NORMAL("In Service"), 
054    DECOMMISSION_INPROGRESS("Decommission In Progress"), 
055    DECOMMISSIONED("Decommissioned");
056
057    final String value;
058
059    AdminStates(final String v) {
060      this.value = v;
061    }
062
063    @Override
064    public String toString() {
065      return value;
066    }
067    
068    public static AdminStates fromValue(final String value) {
069      for (AdminStates as : AdminStates.values()) {
070        if (as.value.equals(value)) return as;
071      }
072      return NORMAL;
073    }
074  }
075
076  protected AdminStates adminState;
077
078  public DatanodeInfo(DatanodeInfo from) {
079    super(from);
080    this.capacity = from.getCapacity();
081    this.dfsUsed = from.getDfsUsed();
082    this.remaining = from.getRemaining();
083    this.blockPoolUsed = from.getBlockPoolUsed();
084    this.lastUpdate = from.getLastUpdate();
085    this.xceiverCount = from.getXceiverCount();
086    this.location = from.getNetworkLocation();
087    this.adminState = from.getAdminState();
088  }
089
090  public DatanodeInfo(DatanodeID nodeID) {
091    super(nodeID);
092    this.capacity = 0L;
093    this.dfsUsed = 0L;
094    this.remaining = 0L;
095    this.blockPoolUsed = 0L;
096    this.lastUpdate = 0L;
097    this.xceiverCount = 0;
098    this.adminState = null;    
099  }
100  
101  public DatanodeInfo(DatanodeID nodeID, String location) {
102    this(nodeID);
103    this.location = location;
104  }
105  
106  public DatanodeInfo(DatanodeID nodeID, String location,
107      final long capacity, final long dfsUsed, final long remaining,
108      final long blockPoolUsed, final long lastUpdate, final int xceiverCount,
109      final AdminStates adminState) {
110    this(nodeID.getIpAddr(), nodeID.getHostName(), nodeID.getStorageID(), nodeID.getXferPort(),
111        nodeID.getInfoPort(), nodeID.getIpcPort(), capacity, dfsUsed, remaining,
112        blockPoolUsed, lastUpdate, xceiverCount, location, adminState);
113  }
114
115  /** Constructor */
116  public DatanodeInfo(final String ipAddr, final String hostName,
117      final String storageID, final int xferPort, final int infoPort, final int ipcPort,
118      final long capacity, final long dfsUsed, final long remaining,
119      final long blockPoolUsed, final long lastUpdate, final int xceiverCount,
120      final String networkLocation, final AdminStates adminState) {
121    super(ipAddr, hostName, storageID, xferPort, infoPort, ipcPort);
122    this.capacity = capacity;
123    this.dfsUsed = dfsUsed;
124    this.remaining = remaining;
125    this.blockPoolUsed = blockPoolUsed;
126    this.lastUpdate = lastUpdate;
127    this.xceiverCount = xceiverCount;
128    this.location = networkLocation;
129    this.adminState = adminState;
130  }
131  
132  /** Network location name */
133  @Override
134  public String getName() {
135    return getXferAddr();
136  }
137  
138  /** The raw capacity. */
139  public long getCapacity() { return capacity; }
140  
141  /** The used space by the data node. */
142  public long getDfsUsed() { return dfsUsed; }
143
144  /** The used space by the block pool on data node. */
145  public long getBlockPoolUsed() { return blockPoolUsed; }
146
147  /** The used space by the data node. */
148  public long getNonDfsUsed() { 
149    long nonDFSUsed = capacity - dfsUsed - remaining;
150    return nonDFSUsed < 0 ? 0 : nonDFSUsed;
151  }
152
153  /** The used space by the data node as percentage of present capacity */
154  public float getDfsUsedPercent() { 
155    return DFSUtil.getPercentUsed(dfsUsed, capacity);
156  }
157
158  /** The raw free space. */
159  public long getRemaining() { return remaining; }
160
161  /** Used space by the block pool as percentage of present capacity */
162  public float getBlockPoolUsedPercent() {
163    return DFSUtil.getPercentUsed(blockPoolUsed, capacity);
164  }
165  
166  /** The remaining space as percentage of configured capacity. */
167  public float getRemainingPercent() { 
168    return DFSUtil.getPercentRemaining(remaining, capacity);
169  }
170
171  /** The time when this information was accurate. */
172  public long getLastUpdate() { return lastUpdate; }
173
174  /** number of active connections */
175  public int getXceiverCount() { return xceiverCount; }
176
177  /** Sets raw capacity. */
178  public void setCapacity(long capacity) { 
179    this.capacity = capacity; 
180  }
181  
182  /** Sets the used space for the datanode. */
183  public void setDfsUsed(long dfsUsed) {
184    this.dfsUsed = dfsUsed;
185  }
186
187  /** Sets raw free space. */
188  public void setRemaining(long remaining) { 
189    this.remaining = remaining; 
190  }
191
192  /** Sets block pool used space */
193  public void setBlockPoolUsed(long bpUsed) { 
194    this.blockPoolUsed = bpUsed; 
195  }
196
197  /** Sets time when this information was accurate. */
198  public void setLastUpdate(long lastUpdate) { 
199    this.lastUpdate = lastUpdate; 
200  }
201
202  /** Sets number of active connections */
203  public void setXceiverCount(int xceiverCount) { 
204    this.xceiverCount = xceiverCount; 
205  }
206
207  /** rack name */
208  public synchronized String getNetworkLocation() {return location;}
209    
210  /** Sets the rack name */
211  public synchronized void setNetworkLocation(String location) {
212    this.location = NodeBase.normalize(location);
213  }
214    
215  /** A formatted string for reporting the status of the DataNode. */
216  public String getDatanodeReport() {
217    StringBuilder buffer = new StringBuilder();
218    long c = getCapacity();
219    long r = getRemaining();
220    long u = getDfsUsed();
221    long nonDFSUsed = getNonDfsUsed();
222    float usedPercent = getDfsUsedPercent();
223    float remainingPercent = getRemainingPercent();
224    String lookupName = NetUtils.getHostNameOfIP(getName());
225
226    buffer.append("Name: "+ getName());
227    if (lookupName != null) {
228      buffer.append(" (" + lookupName + ")");
229    }
230    buffer.append("\n");
231    buffer.append("Hostname: " + getHostName() + "\n");
232
233    if (!NetworkTopology.DEFAULT_RACK.equals(location)) {
234      buffer.append("Rack: "+location+"\n");
235    }
236    buffer.append("Decommission Status : ");
237    if (isDecommissioned()) {
238      buffer.append("Decommissioned\n");
239    } else if (isDecommissionInProgress()) {
240      buffer.append("Decommission in progress\n");
241    } else {
242      buffer.append("Normal\n");
243    }
244    buffer.append("Configured Capacity: "+c+" ("+StringUtils.byteDesc(c)+")"+"\n");
245    buffer.append("DFS Used: "+u+" ("+StringUtils.byteDesc(u)+")"+"\n");
246    buffer.append("Non DFS Used: "+nonDFSUsed+" ("+StringUtils.byteDesc(nonDFSUsed)+")"+"\n");
247    buffer.append("DFS Remaining: " +r+ " ("+StringUtils.byteDesc(r)+")"+"\n");
248    buffer.append("DFS Used%: "+percent2String(usedPercent) + "\n");
249    buffer.append("DFS Remaining%: "+percent2String(remainingPercent) + "\n");
250    buffer.append("Last contact: "+new Date(lastUpdate)+"\n");
251    return buffer.toString();
252  }
253
254  /** A formatted string for printing the status of the DataNode. */
255  public String dumpDatanode() {
256    StringBuilder buffer = new StringBuilder();
257    long c = getCapacity();
258    long r = getRemaining();
259    long u = getDfsUsed();
260    buffer.append(getName());
261    if (!NetworkTopology.DEFAULT_RACK.equals(location)) {
262      buffer.append(" "+location);
263    }
264    if (isDecommissioned()) {
265      buffer.append(" DD");
266    } else if (isDecommissionInProgress()) {
267      buffer.append(" DP");
268    } else {
269      buffer.append(" IN");
270    }
271    buffer.append(" " + c + "(" + StringUtils.byteDesc(c)+")");
272    buffer.append(" " + u + "(" + StringUtils.byteDesc(u)+")");
273    buffer.append(" " + percent2String(u/(double)c));
274    buffer.append(" " + r + "(" + StringUtils.byteDesc(r)+")");
275    buffer.append(" " + new Date(lastUpdate));
276    return buffer.toString();
277  }
278
279  /**
280   * Start decommissioning a node.
281   * old state.
282   */
283  public void startDecommission() {
284    adminState = AdminStates.DECOMMISSION_INPROGRESS;
285  }
286
287  /**
288   * Stop decommissioning a node.
289   * old state.
290   */
291  public void stopDecommission() {
292    adminState = null;
293  }
294
295  /**
296   * Returns true if the node is in the process of being decommissioned
297   */
298  public boolean isDecommissionInProgress() {
299    return adminState == AdminStates.DECOMMISSION_INPROGRESS;
300  }
301
302  /**
303   * Returns true if the node has been decommissioned.
304   */
305  public boolean isDecommissioned() {
306    return adminState == AdminStates.DECOMMISSIONED;
307  }
308
309  /**
310   * Sets the admin state to indicate that decommission is complete.
311   */
312  public void setDecommissioned() {
313    adminState = AdminStates.DECOMMISSIONED;
314  }
315
316  /**
317   * Retrieves the admin state of this node.
318   */
319  public AdminStates getAdminState() {
320    if (adminState == null) {
321      return AdminStates.NORMAL;
322    }
323    return adminState;
324  }
325 
326  /**
327   * Check if the datanode is in stale state. Here if 
328   * the namenode has not received heartbeat msg from a 
329   * datanode for more than staleInterval (default value is
330   * {@link DFSConfigKeys#DFS_NAMENODE_STALE_DATANODE_INTERVAL_MILLI_DEFAULT}),
331   * the datanode will be treated as stale node.
332   * 
333   * @param staleInterval
334   *          the time interval for marking the node as stale. If the last
335   *          update time is beyond the given time interval, the node will be
336   *          marked as stale.
337   * @return true if the node is stale
338   */
339  public boolean isStale(long staleInterval) {
340    return (Time.now() - lastUpdate) >= staleInterval;
341  }
342  
343  /**
344   * Sets the admin state of this node.
345   */
346  protected void setAdminState(AdminStates newState) {
347    if (newState == AdminStates.NORMAL) {
348      adminState = null;
349    }
350    else {
351      adminState = newState;
352    }
353  }
354
355  private transient int level; //which level of the tree the node resides
356  private transient Node parent; //its parent
357
358  /** Return this node's parent */
359  @Override
360  public Node getParent() { return parent; }
361  @Override
362  public void setParent(Node parent) {this.parent = parent;}
363   
364  /** Return this node's level in the tree.
365   * E.g. the root of a tree returns 0 and its children return 1
366   */
367  @Override
368  public int getLevel() { return level; }
369  @Override
370  public void setLevel(int level) {this.level = level;}
371
372  @Override
373  public int hashCode() {
374    // Super implementation is sufficient
375    return super.hashCode();
376  }
377  
378  @Override
379  public boolean equals(Object obj) {
380    // Sufficient to use super equality as datanodes are uniquely identified
381    // by DatanodeID
382    return (this == obj) || super.equals(obj);
383  }
384}