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