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