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 org.apache.hadoop.classification.InterfaceAudience;
021import org.apache.hadoop.classification.InterfaceStability;
022import org.apache.hadoop.hdfs.DFSConfigKeys;
023import org.apache.hadoop.hdfs.DFSUtil;
024import org.apache.hadoop.net.NetUtils;
025import org.apache.hadoop.net.NetworkTopology;
026import org.apache.hadoop.net.Node;
027import org.apache.hadoop.net.NodeBase;
028import org.apache.hadoop.util.StringUtils;
029import org.apache.hadoop.util.Time;
030
031import java.util.Date;
032import java.util.LinkedList;
033import java.util.List;
034
035import static org.apache.hadoop.hdfs.DFSUtil.percent2String;
036
037/** 
038 * This class extends the primary identifier of a Datanode with ephemeral
039 * state, eg usage information, current administrative state, and the
040 * network location that is communicated to clients.
041 */
042@InterfaceAudience.Private
043@InterfaceStability.Evolving
044public class DatanodeInfo extends DatanodeID implements Node {
045  private long capacity;
046  private long dfsUsed;
047  private long nonDfsUsed;
048  private long remaining;
049  private long blockPoolUsed;
050  private long cacheCapacity;
051  private long cacheUsed;
052  private long lastUpdate;
053  private long lastUpdateMonotonic;
054  private int xceiverCount;
055  private String location = NetworkTopology.DEFAULT_RACK;
056  private String softwareVersion;
057  private List<String> dependentHostNames = new LinkedList<String>();
058  
059  
060  // Datanode administrative states
061  public enum AdminStates {
062    NORMAL("In Service"), 
063    DECOMMISSION_INPROGRESS("Decommission In Progress"), 
064    DECOMMISSIONED("Decommissioned");
065
066    final String value;
067
068    AdminStates(final String v) {
069      this.value = v;
070    }
071
072    @Override
073    public String toString() {
074      return value;
075    }
076    
077    public static AdminStates fromValue(final String value) {
078      for (AdminStates as : AdminStates.values()) {
079        if (as.value.equals(value)) return as;
080      }
081      return NORMAL;
082    }
083  }
084
085  protected AdminStates adminState;
086
087  public DatanodeInfo(DatanodeInfo from) {
088    super(from);
089    this.capacity = from.getCapacity();
090    this.dfsUsed = from.getDfsUsed();
091    this.nonDfsUsed = from.getNonDfsUsed();
092    this.remaining = from.getRemaining();
093    this.blockPoolUsed = from.getBlockPoolUsed();
094    this.cacheCapacity = from.getCacheCapacity();
095    this.cacheUsed = from.getCacheUsed();
096    this.lastUpdate = from.getLastUpdate();
097    this.lastUpdateMonotonic = from.getLastUpdateMonotonic();
098    this.xceiverCount = from.getXceiverCount();
099    this.location = from.getNetworkLocation();
100    this.adminState = from.getAdminState();
101  }
102
103  public DatanodeInfo(DatanodeID nodeID) {
104    super(nodeID);
105    this.capacity = 0L;
106    this.dfsUsed = 0L;
107    this.nonDfsUsed = 0L;
108    this.remaining = 0L;
109    this.blockPoolUsed = 0L;
110    this.cacheCapacity = 0L;
111    this.cacheUsed = 0L;
112    this.lastUpdate = 0L;
113    this.lastUpdateMonotonic = 0L;
114    this.xceiverCount = 0;
115    this.adminState = null;    
116  }
117  
118  public DatanodeInfo(DatanodeID nodeID, String location) {
119    this(nodeID);
120    this.location = location;
121  }
122  
123  public DatanodeInfo(DatanodeID nodeID, String location,
124      final long capacity, final long dfsUsed, final long remaining,
125      final long blockPoolUsed, final long cacheCapacity, final long cacheUsed,
126      final long lastUpdate, final long lastUpdateMonotonic,
127      final int xceiverCount, final AdminStates adminState) {
128    this(nodeID.getIpAddr(), nodeID.getHostName(), nodeID.getDatanodeUuid(),
129        nodeID.getXferPort(), nodeID.getInfoPort(), nodeID.getInfoSecurePort(),
130        nodeID.getIpcPort(), capacity, dfsUsed, remaining, blockPoolUsed,
131        cacheCapacity, cacheUsed, lastUpdate, lastUpdateMonotonic,
132        xceiverCount, location, adminState);
133  }
134
135  /** Constructor */
136  public DatanodeInfo(final String ipAddr, final String hostName,
137      final String datanodeUuid, final int xferPort, final int infoPort,
138      final int infoSecurePort, final int ipcPort,
139      final long capacity, final long dfsUsed, final long remaining,
140      final long blockPoolUsed, final long cacheCapacity, final long cacheUsed,
141      final long lastUpdate, final long lastUpdateMonotonic,
142      final int xceiverCount, final String networkLocation,
143      final AdminStates adminState) {
144    this(ipAddr, hostName, datanodeUuid, xferPort, infoPort, infoSecurePort,
145        ipcPort, capacity, dfsUsed, 0L, remaining, blockPoolUsed, cacheCapacity,
146        cacheUsed, lastUpdate, lastUpdateMonotonic, xceiverCount,
147        networkLocation, adminState);
148  }
149  /** Constructor. */
150  public DatanodeInfo(final String ipAddr, final String hostName,
151     final String datanodeUuid, final int xferPort, final int infoPort,
152     final int infoSecurePort, final int ipcPort, final long capacity,
153     final long dfsUsed, final long nonDfsUsed, final long remaining,
154     final long blockPoolUsed, final long cacheCapacity, final long cacheUsed,
155     final long lastUpdate, final long lastUpdateMonotonic,
156     final int xceiverCount, final String networkLocation,
157     final AdminStates adminState) {
158      super(ipAddr, hostName, datanodeUuid, xferPort, infoPort, infoSecurePort,
159                ipcPort);
160    this.capacity = capacity;
161    this.dfsUsed = dfsUsed;
162    this.nonDfsUsed = nonDfsUsed;
163    this.remaining = remaining;
164    this.blockPoolUsed = blockPoolUsed;
165    this.cacheCapacity = cacheCapacity;
166    this.cacheUsed = cacheUsed;
167    this.lastUpdate = lastUpdate;
168    this.lastUpdateMonotonic = lastUpdateMonotonic;
169    this.xceiverCount = xceiverCount;
170    this.location = networkLocation;
171    this.adminState = adminState;
172  }
173  
174  /** Network location name */
175  @Override
176  public String getName() {
177    return getXferAddr();
178  }
179  
180  /** The raw capacity. */
181  public long getCapacity() { return capacity; }
182  
183  /** The used space by the data node. */
184  public long getDfsUsed() { return dfsUsed; }
185
186  /** The used space by the block pool on data node. */
187  public long getBlockPoolUsed() { return blockPoolUsed; }
188
189  /** The used space by the data node. */
190  public long getNonDfsUsed() {
191    return nonDfsUsed;
192  }
193
194  /** The used space by the data node as percentage of present capacity */
195  public float getDfsUsedPercent() { 
196    return DFSUtil.getPercentUsed(dfsUsed, capacity);
197  }
198
199  /** The raw free space. */
200  public long getRemaining() { return remaining; }
201
202  /** Used space by the block pool as percentage of present capacity */
203  public float getBlockPoolUsedPercent() {
204    return DFSUtil.getPercentUsed(blockPoolUsed, capacity);
205  }
206  
207  /** The remaining space as percentage of configured capacity. */
208  public float getRemainingPercent() { 
209    return DFSUtil.getPercentRemaining(remaining, capacity);
210  }
211
212  /**
213   * @return Amount of cache capacity in bytes
214   */
215  public long getCacheCapacity() {
216    return cacheCapacity;
217  }
218
219  /**
220   * @return Amount of cache used in bytes
221   */
222  public long getCacheUsed() {
223    return cacheUsed;
224  }
225
226  /**
227   * @return Cache used as a percentage of the datanode's total cache capacity
228   */
229  public float getCacheUsedPercent() {
230    return DFSUtil.getPercentUsed(cacheUsed, cacheCapacity);
231  }
232
233  /**
234   * @return Amount of cache remaining in bytes
235   */
236  public long getCacheRemaining() {
237    return cacheCapacity - cacheUsed;
238  }
239
240  /**
241   * @return Cache remaining as a percentage of the datanode's total cache
242   * capacity
243   */
244  public float getCacheRemainingPercent() {
245    return DFSUtil.getPercentRemaining(getCacheRemaining(), cacheCapacity);
246  }
247
248  /**
249   * Get the last update timestamp.
250   * Return value is suitable for Date conversion.
251   */
252  public long getLastUpdate() { return lastUpdate; }
253
254  /** 
255   * The time when this information was accurate. <br>
256   * Ps: So return value is ideal for calculation of time differences.
257   * Should not be used to convert to Date.  
258   */
259  public long getLastUpdateMonotonic() { return lastUpdateMonotonic;}
260
261  /**
262   * Set lastUpdate monotonic time
263   */
264  public void setLastUpdateMonotonic(long lastUpdateMonotonic) {
265    this.lastUpdateMonotonic = lastUpdateMonotonic;
266  }
267
268  /** number of active connections */
269  public int getXceiverCount() { return xceiverCount; }
270
271  /** Sets raw capacity. */
272  public void setCapacity(long capacity) { 
273    this.capacity = capacity; 
274  }
275  
276  /** Sets the used space for the datanode. */
277  public void setDfsUsed(long dfsUsed) {
278    this.dfsUsed = dfsUsed;
279  }
280
281  /**
282   * Sets the nondfs-used space for the datanode.
283   */
284  public void setNonDfsUsed(long nonDfsUsed) {
285    this.nonDfsUsed = nonDfsUsed;
286  }
287
288  /** Sets raw free space. */
289  public void setRemaining(long remaining) { 
290    this.remaining = remaining; 
291  }
292
293  /** Sets block pool used space */
294  public void setBlockPoolUsed(long bpUsed) { 
295    this.blockPoolUsed = bpUsed; 
296  }
297
298  /** Sets cache capacity. */
299  public void setCacheCapacity(long cacheCapacity) {
300    this.cacheCapacity = cacheCapacity;
301  }
302
303  /** Sets cache used. */
304  public void setCacheUsed(long cacheUsed) {
305    this.cacheUsed = cacheUsed;
306  }
307
308  /** Sets time when this information was accurate. */
309  public void setLastUpdate(long lastUpdate) { 
310    this.lastUpdate = lastUpdate; 
311  }
312
313  /** Sets number of active connections */
314  public void setXceiverCount(int xceiverCount) { 
315    this.xceiverCount = xceiverCount; 
316  }
317
318  /** network location */
319  public synchronized String getNetworkLocation() {return location;}
320    
321  /** Sets the network location */
322  public synchronized void setNetworkLocation(String location) {
323    this.location = NodeBase.normalize(location);
324  }
325  
326  /** Add a hostname to a list of network dependencies */
327  public void addDependentHostName(String hostname) {
328    dependentHostNames.add(hostname);
329  }
330  
331  /** List of Network dependencies */
332  public List<String> getDependentHostNames() {
333    return dependentHostNames;
334  }
335  
336  /** Sets the network dependencies */
337  public void setDependentHostNames(List<String> dependencyList) {
338    dependentHostNames = dependencyList;
339  }
340    
341  /** A formatted string for reporting the status of the DataNode. */
342  public String getDatanodeReport() {
343    StringBuilder buffer = new StringBuilder();
344    long c = getCapacity();
345    long r = getRemaining();
346    long u = getDfsUsed();
347    long nonDFSUsed = getNonDfsUsed();
348    float usedPercent = getDfsUsedPercent();
349    float remainingPercent = getRemainingPercent();
350    long cc = getCacheCapacity();
351    long cr = getCacheRemaining();
352    long cu = getCacheUsed();
353    float cacheUsedPercent = getCacheUsedPercent();
354    float cacheRemainingPercent = getCacheRemainingPercent();
355    String lookupName = NetUtils.getHostNameOfIP(getName());
356
357    buffer.append("Name: "+ getName());
358    if (lookupName != null) {
359      buffer.append(" (" + lookupName + ")");
360    }
361    buffer.append("\n");
362    buffer.append("Hostname: " + getHostName() + "\n");
363
364    if (!NetworkTopology.DEFAULT_RACK.equals(location)) {
365      buffer.append("Rack: "+location+"\n");
366    }
367    buffer.append("Decommission Status : ");
368    if (isDecommissioned()) {
369      buffer.append("Decommissioned\n");
370    } else if (isDecommissionInProgress()) {
371      buffer.append("Decommission in progress\n");
372    } else {
373      buffer.append("Normal\n");
374    }
375    buffer.append("Configured Capacity: "+c+" ("+StringUtils.byteDesc(c)+")"+"\n");
376    buffer.append("DFS Used: "+u+" ("+StringUtils.byteDesc(u)+")"+"\n");
377    buffer.append("Non DFS Used: "+nonDFSUsed+" ("+StringUtils.byteDesc(nonDFSUsed)+")"+"\n");
378    buffer.append("DFS Remaining: " +r+ " ("+StringUtils.byteDesc(r)+")"+"\n");
379    buffer.append("DFS Used%: "+percent2String(usedPercent) + "\n");
380    buffer.append("DFS Remaining%: "+percent2String(remainingPercent) + "\n");
381    buffer.append("Configured Cache Capacity: "+cc+" ("+StringUtils.byteDesc(cc)+")"+"\n");
382    buffer.append("Cache Used: "+cu+" ("+StringUtils.byteDesc(cu)+")"+"\n");
383    buffer.append("Cache Remaining: " +cr+ " ("+StringUtils.byteDesc(cr)+")"+"\n");
384    buffer.append("Cache Used%: "+percent2String(cacheUsedPercent) + "\n");
385    buffer.append("Cache Remaining%: "+percent2String(cacheRemainingPercent) + "\n");
386    buffer.append("Xceivers: "+getXceiverCount()+"\n");
387    buffer.append("Last contact: "+new Date(lastUpdate)+"\n");
388    return buffer.toString();
389  }
390
391  /** A formatted string for printing the status of the DataNode. */
392  public String dumpDatanode() {
393    StringBuilder buffer = new StringBuilder();
394    long c = getCapacity();
395    long r = getRemaining();
396    long u = getDfsUsed();
397    float usedPercent = getDfsUsedPercent();
398    long cc = getCacheCapacity();
399    long cr = getCacheRemaining();
400    long cu = getCacheUsed();
401    float cacheUsedPercent = getCacheUsedPercent();
402    buffer.append(getName());
403    if (!NetworkTopology.DEFAULT_RACK.equals(location)) {
404      buffer.append(" "+location);
405    }
406    if (isDecommissioned()) {
407      buffer.append(" DD");
408    } else if (isDecommissionInProgress()) {
409      buffer.append(" DP");
410    } else {
411      buffer.append(" IN");
412    }
413    buffer.append(" " + c + "(" + StringUtils.byteDesc(c)+")");
414    buffer.append(" " + u + "(" + StringUtils.byteDesc(u)+")");
415    buffer.append(" " + percent2String(usedPercent));
416    buffer.append(" " + r + "(" + StringUtils.byteDesc(r)+")");
417    buffer.append(" " + cc + "(" + StringUtils.byteDesc(cc)+")");
418    buffer.append(" " + cu + "(" + StringUtils.byteDesc(cu)+")");
419    buffer.append(" " + percent2String(cacheUsedPercent));
420    buffer.append(" " + cr + "(" + StringUtils.byteDesc(cr)+")");
421    buffer.append(" " + new Date(lastUpdate));
422    return buffer.toString();
423  }
424
425  /**
426   * Start decommissioning a node.
427   * old state.
428   */
429  public void startDecommission() {
430    adminState = AdminStates.DECOMMISSION_INPROGRESS;
431  }
432
433  /**
434   * Stop decommissioning a node.
435   * old state.
436   */
437  public void stopDecommission() {
438    adminState = null;
439  }
440
441  /**
442   * Returns true if the node is in the process of being decommissioned
443   */
444  public boolean isDecommissionInProgress() {
445    return adminState == AdminStates.DECOMMISSION_INPROGRESS;
446  }
447
448  /**
449   * Returns true if the node has been decommissioned.
450   */
451  public boolean isDecommissioned() {
452    return adminState == AdminStates.DECOMMISSIONED;
453  }
454
455  /**
456   * Sets the admin state to indicate that decommission is complete.
457   */
458  public void setDecommissioned() {
459    adminState = AdminStates.DECOMMISSIONED;
460  }
461
462  /**
463   * Retrieves the admin state of this node.
464   */
465  public AdminStates getAdminState() {
466    if (adminState == null) {
467      return AdminStates.NORMAL;
468    }
469    return adminState;
470  }
471 
472  /**
473   * Check if the datanode is in stale state. Here if 
474   * the namenode has not received heartbeat msg from a 
475   * datanode for more than staleInterval (default value is
476   * {@link DFSConfigKeys#DFS_NAMENODE_STALE_DATANODE_INTERVAL_DEFAULT}),
477   * the datanode will be treated as stale node.
478   * 
479   * @param staleInterval
480   *          the time interval for marking the node as stale. If the last
481   *          update time is beyond the given time interval, the node will be
482   *          marked as stale.
483   * @return true if the node is stale
484   */
485  public boolean isStale(long staleInterval) {
486    return (Time.monotonicNow() - lastUpdateMonotonic) >= staleInterval;
487  }
488  
489  /**
490   * Sets the admin state of this node.
491   */
492  protected void setAdminState(AdminStates newState) {
493    if (newState == AdminStates.NORMAL) {
494      adminState = null;
495    }
496    else {
497      adminState = newState;
498    }
499  }
500
501  private transient int level; //which level of the tree the node resides
502  private transient Node parent; //its parent
503
504  /** Return this node's parent */
505  @Override
506  public Node getParent() { return parent; }
507  @Override
508  public void setParent(Node parent) {this.parent = parent;}
509   
510  /** Return this node's level in the tree.
511   * E.g. the root of a tree returns 0 and its children return 1
512   */
513  @Override
514  public int getLevel() { return level; }
515  @Override
516  public void setLevel(int level) {this.level = level;}
517
518  @Override
519  public int hashCode() {
520    // Super implementation is sufficient
521    return super.hashCode();
522  }
523  
524  @Override
525  public boolean equals(Object obj) {
526    // Sufficient to use super equality as datanodes are uniquely identified
527    // by DatanodeID
528    return (this == obj) || super.equals(obj);
529  }
530
531  public String getSoftwareVersion() {
532    return softwareVersion;
533  }
534
535  public void setSoftwareVersion(String softwareVersion) {
536    this.softwareVersion = softwareVersion;
537  }
538}