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 java.io.DataInput;
021import java.io.DataOutput;
022import java.io.IOException;
023import java.util.Date;
024
025import org.apache.hadoop.classification.InterfaceAudience;
026import org.apache.hadoop.classification.InterfaceStability;
027import org.apache.hadoop.hdfs.DFSUtil;
028import org.apache.hadoop.io.Text;
029import org.apache.hadoop.io.Writable;
030import org.apache.hadoop.io.WritableFactories;
031import org.apache.hadoop.io.WritableFactory;
032import org.apache.hadoop.io.WritableUtils;
033import org.apache.hadoop.net.NetUtils;
034import org.apache.hadoop.net.NetworkTopology;
035import org.apache.hadoop.net.Node;
036import org.apache.hadoop.net.NodeBase;
037import org.apache.hadoop.util.StringUtils;
038
039import org.apache.avro.reflect.Nullable;
040
041/** 
042 * DatanodeInfo represents the status of a DataNode.
043 * This object is used for communication in the
044 * Datanode Protocol and the Client Protocol.
045 */
046@InterfaceAudience.Private
047@InterfaceStability.Evolving
048public class DatanodeInfo extends DatanodeID implements Node {
049  protected long capacity;
050  protected long dfsUsed;
051  protected long remaining;
052  protected long blockPoolUsed;
053  protected long lastUpdate;
054  protected int xceiverCount;
055  protected String location = NetworkTopology.DEFAULT_RACK;
056
057  /** HostName as supplied by the datanode during registration as its 
058   * name. Namenode uses datanode IP address as the name.
059   */
060  @Nullable
061  protected String hostName = null;
062  
063  // administrative states of a datanode
064  public enum AdminStates {
065    NORMAL("In Service"), 
066    DECOMMISSION_INPROGRESS("Decommission In Progress"), 
067    DECOMMISSIONED("Decommissioned");
068
069    final String value;
070
071    AdminStates(final String v) {
072      this.value = v;
073    }
074
075    public String toString() {
076      return value;
077    }
078  }
079
080  @Nullable
081  protected AdminStates adminState;
082
083
084  public DatanodeInfo() {
085    super();
086    adminState = null;
087  }
088  
089  public DatanodeInfo(DatanodeInfo from) {
090    super(from);
091    this.capacity = from.getCapacity();
092    this.dfsUsed = from.getDfsUsed();
093    this.remaining = from.getRemaining();
094    this.blockPoolUsed = from.getBlockPoolUsed();
095    this.lastUpdate = from.getLastUpdate();
096    this.xceiverCount = from.getXceiverCount();
097    this.location = from.getNetworkLocation();
098    this.adminState = from.adminState;
099    this.hostName = from.hostName;
100  }
101
102  public DatanodeInfo(DatanodeID nodeID) {
103    super(nodeID);
104    this.capacity = 0L;
105    this.dfsUsed = 0L;
106    this.remaining = 0L;
107    this.blockPoolUsed = 0L;
108    this.lastUpdate = 0L;
109    this.xceiverCount = 0;
110    this.adminState = null;    
111  }
112  
113  protected DatanodeInfo(DatanodeID nodeID, String location, String hostName) {
114    this(nodeID);
115    this.location = location;
116    this.hostName = hostName;
117  }
118
119  /** Constructor */
120  public DatanodeInfo(final String name, final String storageID,
121      final int infoPort, final int ipcPort,
122      final long capacity, final long dfsUsed, final long remaining,
123      final long blockPoolUsed, final long lastUpdate, final int xceiverCount,
124      final String networkLocation, final String hostName,
125      final AdminStates adminState) {
126    super(name, storageID, infoPort, ipcPort);
127
128    this.capacity = capacity;
129    this.dfsUsed = dfsUsed;
130    this.remaining = remaining;
131    this.blockPoolUsed = blockPoolUsed;
132    this.lastUpdate = lastUpdate;
133    this.xceiverCount = xceiverCount;
134    this.location = networkLocation;
135    this.hostName = hostName;
136    this.adminState = adminState;
137  }
138  
139  /** The raw capacity. */
140  public long getCapacity() { return capacity; }
141  
142  /** The used space by the data node. */
143  public long getDfsUsed() { return dfsUsed; }
144
145  /** The used space by the block pool on data node. */
146  public long getBlockPoolUsed() { return blockPoolUsed; }
147
148  /** The used space by the data node. */
149  public long getNonDfsUsed() { 
150    long nonDFSUsed = capacity - dfsUsed - remaining;
151    return nonDFSUsed < 0 ? 0 : nonDFSUsed;
152  }
153
154  /** The used space by the data node as percentage of present capacity */
155  public float getDfsUsedPercent() { 
156    return DFSUtil.getPercentUsed(dfsUsed, capacity);
157  }
158
159  /** The raw free space. */
160  public long getRemaining() { return remaining; }
161
162  /** Used space by the block pool as percentage of present capacity */
163  public float getBlockPoolUsedPercent() {
164    return DFSUtil.getPercentUsed(blockPoolUsed, capacity);
165  }
166  
167  /** The remaining space as percentage of configured capacity. */
168  public float getRemainingPercent() { 
169    return DFSUtil.getPercentRemaining(remaining, capacity);
170  }
171
172  /** The time when this information was accurate. */
173  public long getLastUpdate() { return lastUpdate; }
174
175  /** number of active connections */
176  public int getXceiverCount() { return xceiverCount; }
177
178  /** Sets raw capacity. */
179  public void setCapacity(long capacity) { 
180    this.capacity = capacity; 
181  }
182  
183  /** Sets the used space for the datanode. */
184  public void setDfsUsed(long dfsUsed) {
185    this.dfsUsed = dfsUsed;
186  }
187
188  /** Sets raw free space. */
189  public void setRemaining(long remaining) { 
190    this.remaining = remaining; 
191  }
192
193  /** Sets block pool used space */
194  public void setBlockPoolUsed(long bpUsed) { 
195    this.blockPoolUsed = bpUsed; 
196  }
197
198  /** Sets time when this information was accurate. */
199  public void setLastUpdate(long lastUpdate) { 
200    this.lastUpdate = lastUpdate; 
201  }
202
203  /** Sets number of active connections */
204  public void setXceiverCount(int xceiverCount) { 
205    this.xceiverCount = xceiverCount; 
206  }
207
208  /** rack name */
209  public synchronized String getNetworkLocation() {return location;}
210    
211  /** Sets the rack name */
212  public synchronized void setNetworkLocation(String location) {
213    this.location = NodeBase.normalize(location);
214  }
215  
216  public String getHostName() {
217    return (hostName == null || hostName.length()==0) ? getHost() : hostName;
218  }
219  
220  public void setHostName(String host) {
221    hostName = host;
222  }
223  
224  /** A formatted string for reporting the status of the DataNode. */
225  public String getDatanodeReport() {
226    StringBuilder buffer = new StringBuilder();
227    long c = getCapacity();
228    long r = getRemaining();
229    long u = getDfsUsed();
230    long nonDFSUsed = getNonDfsUsed();
231    float usedPercent = getDfsUsedPercent();
232    float remainingPercent = getRemainingPercent();
233    String hostName = NetUtils.getHostNameOfIP(name);
234
235    buffer.append("Name: "+ name);
236    if(hostName != null)
237      buffer.append(" (" + hostName + ")");
238    buffer.append("\n");
239
240    if (!NetworkTopology.DEFAULT_RACK.equals(location)) {
241      buffer.append("Rack: "+location+"\n");
242    }
243    buffer.append("Decommission Status : ");
244    if (isDecommissioned()) {
245      buffer.append("Decommissioned\n");
246    } else if (isDecommissionInProgress()) {
247      buffer.append("Decommission in progress\n");
248    } else {
249      buffer.append("Normal\n");
250    }
251    buffer.append("Configured Capacity: "+c+" ("+StringUtils.byteDesc(c)+")"+"\n");
252    buffer.append("DFS Used: "+u+" ("+StringUtils.byteDesc(u)+")"+"\n");
253    buffer.append("Non DFS Used: "+nonDFSUsed+" ("+StringUtils.byteDesc(nonDFSUsed)+")"+"\n");
254    buffer.append("DFS Remaining: " +r+ " ("+StringUtils.byteDesc(r)+")"+"\n");
255    buffer.append("DFS Used%: "+StringUtils.limitDecimalTo2(usedPercent)+"%\n");
256    buffer.append("DFS Remaining%: "+StringUtils.limitDecimalTo2(remainingPercent)+"%\n");
257    buffer.append("Last contact: "+new Date(lastUpdate)+"\n");
258    return buffer.toString();
259  }
260
261  /** A formatted string for printing the status of the DataNode. */
262  public String dumpDatanode() {
263    StringBuilder buffer = new StringBuilder();
264    long c = getCapacity();
265    long r = getRemaining();
266    long u = getDfsUsed();
267    buffer.append(name);
268    if (!NetworkTopology.DEFAULT_RACK.equals(location)) {
269      buffer.append(" "+location);
270    }
271    if (isDecommissioned()) {
272      buffer.append(" DD");
273    } else if (isDecommissionInProgress()) {
274      buffer.append(" DP");
275    } else {
276      buffer.append(" IN");
277    }
278    buffer.append(" " + c + "(" + StringUtils.byteDesc(c)+")");
279    buffer.append(" " + u + "(" + StringUtils.byteDesc(u)+")");
280    buffer.append(" " + StringUtils.limitDecimalTo2(((1.0*u)/c)*100)+"%");
281    buffer.append(" " + r + "(" + StringUtils.byteDesc(r)+")");
282    buffer.append(" " + new Date(lastUpdate));
283    return buffer.toString();
284  }
285
286  /**
287   * Start decommissioning a node.
288   * old state.
289   */
290  public void startDecommission() {
291    adminState = AdminStates.DECOMMISSION_INPROGRESS;
292  }
293
294  /**
295   * Stop decommissioning a node.
296   * old state.
297   */
298  public void stopDecommission() {
299    adminState = null;
300  }
301
302  /**
303   * Returns true if the node is in the process of being decommissioned
304   */
305  public boolean isDecommissionInProgress() {
306    return adminState == AdminStates.DECOMMISSION_INPROGRESS;
307  }
308
309  /**
310   * Returns true if the node has been decommissioned.
311   */
312  public boolean isDecommissioned() {
313    return adminState == AdminStates.DECOMMISSIONED;
314  }
315
316  /**
317   * Sets the admin state to indicate that decommission is complete.
318   */
319  public void setDecommissioned() {
320    adminState = AdminStates.DECOMMISSIONED;
321  }
322
323  /**
324   * Retrieves the admin state of this node.
325   */
326  public AdminStates getAdminState() {
327    if (adminState == null) {
328      return AdminStates.NORMAL;
329    }
330    return adminState;
331  }
332
333  /**
334   * Sets the admin state of this node.
335   */
336  protected void setAdminState(AdminStates newState) {
337    if (newState == AdminStates.NORMAL) {
338      adminState = null;
339    }
340    else {
341      adminState = newState;
342    }
343  }
344
345  private transient int level; //which level of the tree the node resides
346  private transient Node parent; //its parent
347
348  /** Return this node's parent */
349  public Node getParent() { return parent; }
350  public void setParent(Node parent) {this.parent = parent;}
351   
352  /** Return this node's level in the tree.
353   * E.g. the root of a tree returns 0 and its children return 1
354   */
355  public int getLevel() { return level; }
356  public void setLevel(int level) {this.level = level;}
357
358  /////////////////////////////////////////////////
359  // Writable
360  /////////////////////////////////////////////////
361  static {                                      // register a ctor
362    WritableFactories.setFactory
363      (DatanodeInfo.class,
364       new WritableFactory() {
365         public Writable newInstance() { return new DatanodeInfo(); }
366       });
367  }
368
369  /** {@inheritDoc} */
370  public void write(DataOutput out) throws IOException {
371    super.write(out);
372
373    //TODO: move it to DatanodeID once DatanodeID is not stored in FSImage
374    out.writeShort(ipcPort);
375
376    out.writeLong(capacity);
377    out.writeLong(dfsUsed);
378    out.writeLong(remaining);
379    out.writeLong(blockPoolUsed);
380    out.writeLong(lastUpdate);
381    out.writeInt(xceiverCount);
382    Text.writeString(out, location);
383    Text.writeString(out, hostName == null? "": hostName);
384    WritableUtils.writeEnum(out, getAdminState());
385  }
386
387  /** {@inheritDoc} */
388  public void readFields(DataInput in) throws IOException {
389    super.readFields(in);
390
391    //TODO: move it to DatanodeID once DatanodeID is not stored in FSImage
392    this.ipcPort = in.readShort() & 0x0000ffff;
393
394    this.capacity = in.readLong();
395    this.dfsUsed = in.readLong();
396    this.remaining = in.readLong();
397    this.blockPoolUsed = in.readLong();
398    this.lastUpdate = in.readLong();
399    this.xceiverCount = in.readInt();
400    this.location = Text.readString(in);
401    this.hostName = Text.readString(in);
402    setAdminState(WritableUtils.readEnum(in, AdminStates.class));
403  }
404
405  /** Read a DatanodeInfo */
406  public static DatanodeInfo read(DataInput in) throws IOException {
407    final DatanodeInfo d = new DatanodeInfo();
408    d.readFields(in);
409    return d;
410  }
411
412  @Override
413  public int hashCode() {
414    // Super implementation is sufficient
415    return super.hashCode();
416  }
417  
418  @Override
419  public boolean equals(Object obj) {
420    // Sufficient to use super equality as datanodes are uniquely identified
421    // by DatanodeID
422    return (this == obj) || super.equals(obj);
423  }
424}