You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by dh...@apache.org on 2007/09/10 20:46:44 UTC

svn commit: r574331 - in /lucene/hadoop/trunk: ./ src/java/org/apache/hadoop/dfs/ src/java/org/apache/hadoop/net/ src/test/org/apache/hadoop/dfs/

Author: dhruba
Date: Mon Sep 10 11:46:43 2007
New Revision: 574331

URL: http://svn.apache.org/viewvc?rev=574331&view=rev
Log:
HADOOP-1266. Remove dependency of package org.apache.hadoop.net on
org.apache.hadoop.dfs.  (Hairong Kuang via dhruba)


Modified:
    lucene/hadoop/trunk/CHANGES.txt
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeInfo.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/ReplicationTargetChooser.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NetworkTopology.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/net/Node.java
    lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NodeBase.java
    lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestReplicationPolicy.java

Modified: lucene/hadoop/trunk/CHANGES.txt
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/CHANGES.txt?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/CHANGES.txt (original)
+++ lucene/hadoop/trunk/CHANGES.txt Mon Sep 10 11:46:43 2007
@@ -110,6 +110,9 @@
 
   IMPROVEMENTS
 
+    HADOOP-1266. Remove dependency of package org.apache.hadoop.net on 
+    org.apache.hadoop.dfs.  (Hairong Kuang via dhruba)
+
     HADOOP-1779. Replace INodeDirectory.getINode() by a getExistingPathINodes()
     to allow the retrieval of all existing INodes along a given path in a
     single lookup. This facilitates removal of the 'parent' field in the

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeInfo.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeInfo.java?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeInfo.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/DatanodeInfo.java Mon Sep 10 11:46:43 2007
@@ -132,11 +132,6 @@
     this.location = NodeBase.normalize(location);
   }
   
-  public String getPath() {
-    return location+NodeBase.PATH_SEPARATOR_STR+name;
-  }
-
-  
   public String getHostName() {
     return (hostName == null || hostName.length()==0) ? getHost() : hostName;
   }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/ReplicationTargetChooser.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/ReplicationTargetChooser.java?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/ReplicationTargetChooser.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/dfs/ReplicationTargetChooser.java Mon Sep 10 11:46:43 2007
@@ -19,6 +19,7 @@
 
 import org.apache.commons.logging.*;
 import org.apache.hadoop.net.NetworkTopology;
+import org.apache.hadoop.net.Node;
 import org.apache.hadoop.net.NodeBase;
 import java.util.*;
 
@@ -62,10 +63,10 @@
    */
   DatanodeDescriptor[] chooseTarget(int numOfReplicas,
                                     DatanodeDescriptor writer,
-                                    List<DatanodeDescriptor> excludedNodes,
+                                    List<Node> excludedNodes,
                                     long blocksize) {
     if (excludedNodes == null) {
-      excludedNodes = new ArrayList<DatanodeDescriptor>();
+      excludedNodes = new ArrayList<Node>();
     }
       
     return chooseTarget(numOfReplicas, writer, 
@@ -88,14 +89,14 @@
   DatanodeDescriptor[] chooseTarget(int numOfReplicas,
                                     DatanodeDescriptor writer,
                                     List<DatanodeDescriptor> choosenNodes,
-                                    List<DatanodeDescriptor> excludedNodes,
+                                    List<Node> excludedNodes,
                                     long blocksize) {
     if (numOfReplicas == 0 || clusterMap.getNumOfLeaves()==0) {
       return new DatanodeDescriptor[0];
     }
       
     if (excludedNodes == null) {
-      excludedNodes = new ArrayList<DatanodeDescriptor>();
+      excludedNodes = new ArrayList<Node>();
     }
       
     int clusterSize = clusterMap.getNumOfLeaves();
@@ -129,7 +130,7 @@
   /* choose <i>numOfReplicas</i> from all data nodes */
   private DatanodeDescriptor chooseTarget(int numOfReplicas,
                                           DatanodeDescriptor writer,
-                                          List<DatanodeDescriptor> excludedNodes,
+                                          List<Node> excludedNodes,
                                           long blocksize,
                                           int maxNodesPerRack,
                                           List<DatanodeDescriptor> results) {
@@ -140,7 +141,7 @@
       
     int numOfResults = results.size();
     if (writer == null && (numOfResults==1 || numOfResults==2)) {
-      writer = results.get(0);
+      writer = (DatanodeDescriptor)results.get(0);
     }
       
     try {
@@ -186,7 +187,7 @@
    */
   private DatanodeDescriptor chooseLocalNode(
                                              DatanodeDescriptor localMachine,
-                                             List<DatanodeDescriptor> excludedNodes,
+                                             List<Node> excludedNodes,
                                              long blocksize,
                                              int maxNodesPerRack,
                                              List<DatanodeDescriptor> results)
@@ -220,7 +221,7 @@
    */
   private DatanodeDescriptor chooseLocalRack(
                                              DatanodeDescriptor localMachine,
-                                             List<DatanodeDescriptor> excludedNodes,
+                                             List<Node> excludedNodes,
                                              long blocksize,
                                              int maxNodesPerRack,
                                              List<DatanodeDescriptor> results)
@@ -273,7 +274,7 @@
     
   private void chooseRemoteRack(int numOfReplicas,
                                 DatanodeDescriptor localMachine,
-                                List<DatanodeDescriptor> excludedNodes,
+                                List<Node> excludedNodes,
                                 long blocksize,
                                 int maxReplicasPerRack,
                                 List<DatanodeDescriptor> results)
@@ -295,7 +296,7 @@
    */
   private DatanodeDescriptor chooseRandom(
                                           String nodes,
-                                          List<DatanodeDescriptor> excludedNodes,
+                                          List<Node> excludedNodes,
                                           long blocksize,
                                           int maxNodesPerRack,
                                           List<DatanodeDescriptor> results) 
@@ -318,7 +319,7 @@
    */
   private void chooseRandom(int numOfReplicas,
                             String nodes,
-                            List<DatanodeDescriptor> excludedNodes,
+                            List<Node> excludedNodes,
                             long blocksize,
                             int maxNodesPerRack,
                             List<DatanodeDescriptor> results)
@@ -331,7 +332,7 @@
         toContinue = false;
       }
       for(int i=0; i<selectedNodes.length; i++) {
-        DatanodeDescriptor result = (DatanodeDescriptor)(selectedNodes[i]);
+        DatanodeDescriptor result = selectedNodes[i];
         if (isGoodTarget(result, blocksize, maxNodesPerRack, results)) {
           numOfReplicas--;
           results.add(result);
@@ -350,7 +351,7 @@
    */
   private DatanodeDescriptor[] chooseRandom(int numOfReplicas, 
                                             String nodes,
-                                            List<DatanodeDescriptor> excludedNodes) {
+                                            List<Node> excludedNodes) {
     List<DatanodeDescriptor> results = 
       new ArrayList<DatanodeDescriptor>();
     int numOfAvailableNodes =
@@ -358,7 +359,8 @@
     numOfReplicas = (numOfAvailableNodes<numOfReplicas)?
       numOfAvailableNodes:numOfReplicas;
     while(numOfReplicas > 0) {
-      DatanodeDescriptor choosenNode = clusterMap.chooseRandom(nodes);
+      DatanodeDescriptor choosenNode = 
+        (DatanodeDescriptor)(clusterMap.chooseRandom(nodes));
       if (!excludedNodes.contains(choosenNode)) {
         results.add(choosenNode);
         excludedNodes.add(choosenNode);
@@ -387,14 +389,14 @@
     Log logr = FSNamesystem.LOG;
     // check if the node is (being) decommissed
     if (node.isDecommissionInProgress() || node.isDecommissioned()) {
-      logr.debug("Node "+node.getPath()+
+      logr.debug("Node "+NodeBase.getPath(node)+
                 " is not chosen because the node is (being) decommissioned");
       return false;
     }
 
     // check the remaining capacity of the target machine
     if (blockSize* FSConstants.MIN_BLOCKS_FOR_WRITE>node.getRemaining()) {
-      logr.debug("Node "+node.getPath()+
+      logr.debug("Node "+NodeBase.getPath(node)+
                 " is not chosen because the node does not have enough space");
       return false;
     }
@@ -407,7 +409,7 @@
         avgLoad = (double)fs.totalLoad()/size;
       }
       if (node.getXceiverCount() > (2.0 * avgLoad)) {
-        logr.debug("Node "+node.getPath()+
+        logr.debug("Node "+NodeBase.getPath(node)+
                   " is not chosen because the node is too busy");
         return false;
       }
@@ -418,13 +420,13 @@
     int counter=1;
     for(Iterator<DatanodeDescriptor> iter = results.iterator();
         iter.hasNext();) {
-      DatanodeDescriptor result = iter.next();
+      Node result = iter.next();
       if (rackname.equals(result.getNetworkLocation())) {
         counter++;
       }
     }
     if (counter>maxTargetPerLoc) {
-      logr.debug("Node "+node.getPath()+
+      logr.debug("Node "+NodeBase.getPath(node)+
                 " is not chosen because the rack has too many chosen nodes");
       return false;
     }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NetworkTopology.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NetworkTopology.java?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NetworkTopology.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NetworkTopology.java Mon Sep 10 11:46:43 2007
@@ -26,7 +26,6 @@
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
-import org.apache.hadoop.dfs.DatanodeDescriptor;
 
 /** The class represents a cluster of computer with a tree hierarchical
  * network topology.
@@ -95,9 +94,9 @@
      * @return true if this node is an ancestor of <i>n</i>
      */
     boolean isAncestor(Node n) {
-      return getPath().equals(NodeBase.PATH_SEPARATOR_STR) ||
+      return getPath(this).equals(NodeBase.PATH_SEPARATOR_STR) ||
         (n.getNetworkLocation()+NodeBase.PATH_SEPARATOR_STR).
-        startsWith(getPath()+NodeBase.PATH_SEPARATOR_STR);
+        startsWith(getPath(this)+NodeBase.PATH_SEPARATOR_STR);
     }
         
     /** Judge if this node is the parent of node <i>n</i>
@@ -106,7 +105,7 @@
      * @return true if this node is the parent of <i>n</i>
      */
     boolean isParent(Node n) {
-      return n.getNetworkLocation().equals(getPath());
+      return n.getNetworkLocation().equals(getPath(this));
     }
         
     /* Return a child name of this node who is an ancestor of node <i>n</i> */
@@ -115,7 +114,7 @@
         throw new IllegalArgumentException(
                                            this + "is not an ancestor of " + n);
       }
-      String name = n.getNetworkLocation().substring(getPath().length());
+      String name = n.getNetworkLocation().substring(getPath(this).length());
       if (name.charAt(0) == PATH_SEPARATOR) {
         name = name.substring(1);
       }
@@ -129,10 +128,11 @@
      * @param n node to be added
      * @return true if the node is added; false otherwise
      */
-    boolean add(DatanodeDescriptor n) {
+    boolean add(Node n) {
       if (!isAncestor(n))
         throw new IllegalArgumentException(n.getName()+", which is located at "
-                                           +n.getNetworkLocation()+", is not a decendent of "+getPath());
+                +n.getNetworkLocation()+", is not a decendent of "
+                +getPath(this));
       if (isParent(n)) {
         // this node is the parent of n; add n directly
         n.setParent(this);
@@ -158,7 +158,7 @@
         }
         if (parentNode == null) {
           // create a new InnerNode
-          parentNode = new InnerNode(parentName, getPath(),
+          parentNode = new InnerNode(parentName, getPath(this),
                                      this, this.getLevel()+1);
           children.add(parentNode);
         }
@@ -176,9 +176,9 @@
      * @parameter n node to be deleted 
      * @return true if the node is deleted; false otherwise
      */
-    boolean remove(DatanodeDescriptor n) {
+    boolean remove(Node n) {
       String parent = n.getNetworkLocation();
-      String currentPath = getPath();
+      String currentPath = getPath(this);
       if (!isAncestor(n))
         throw new IllegalArgumentException(n.getName()
                                            +", which is located at "
@@ -243,7 +243,7 @@
         
     /** get <i>leaveIndex</i> leaf of this subtree 
      * if it is not in the <i>excludedNode</i>*/
-    private DatanodeDescriptor getLeaf(int leaveIndex, Node excludedNode) {
+    private Node getLeaf(int leaveIndex, Node excludedNode) {
       int count=0;
       int numOfExcludedLeaves = 1;
       if (excludedNode instanceof InnerNode) {
@@ -254,8 +254,7 @@
         if (leaveIndex<0 || leaveIndex>=this.getNumOfChildren()) {
           return null;
         }
-        DatanodeDescriptor child =
-          (DatanodeDescriptor)children.get(leaveIndex);
+        Node child = children.get(leaveIndex);
         if (excludedNode == null || excludedNode != child) {
           // child is not the excludedNode
           return child;
@@ -263,7 +262,7 @@
           if (leaveIndex+1>=this.getNumOfChildren()) {
             return null;
           } else {
-            return (DatanodeDescriptor)children.get(leaveIndex+1);
+            return children.get(leaveIndex+1);
           }
         }
       } else {
@@ -304,16 +303,21 @@
     netlock = new ReentrantReadWriteLock();
   }
     
-  /** Add a data node
-   * Update data node counter & rack counter if neccessary
+  /** Add a leaf node
+   * Update node counter & rack counter if neccessary
    * @param node
-   *          data node to be added
-   * @exception IllegalArgumentException if add a data node to a leave
+   *          node to be added
+   * @exception IllegalArgumentException if add a node to a leave 
+                                         or node to be added is not a leaf
    */
-  public void add(DatanodeDescriptor node) {
+  public void add(Node node) {
     if (node==null) return;
+    if( node instanceof InnerNode ) {
+      throw new IllegalArgumentException(
+        "Not allow to add an inner node: "+NodeBase.getPath(node));
+    }
     netlock.writeLock().lock();
-    LOG.info("Adding a new node: "+node.getPath());
+    LOG.info("Adding a new node: "+NodeBase.getPath(node));
     try {
       Node rack = getNode(node.getNetworkLocation());
       if (rack != null && !(rack instanceof InnerNode)) {
@@ -332,15 +336,19 @@
     }
   }
     
-  /** Remove a data node
-   * Update data node counter & rack counter if neccessary
+  /** Remove a node
+   * Update node counter & rack counter if neccessary
    * @param node
-   *          data node to be removed
+   *          node to be removed
    */ 
-  public void remove(DatanodeDescriptor node) {
+  public void remove(Node node) {
     if (node==null) return;
     netlock.writeLock().lock();
-    LOG.info("Removing a node: "+node.getPath());
+    if( node instanceof InnerNode ) {
+      throw new IllegalArgumentException(
+        "Not allow to remove an inner node: "+NodeBase.getPath(node));
+    }
+    LOG.info("Removing a node: "+NodeBase.getPath(node));
     try {
       if (clusterMap.remove(node)) {
         InnerNode rack = (InnerNode)getNode(node.getNetworkLocation());
@@ -354,13 +362,13 @@
     }
   }
        
-  /** Check if the tree contains data node <i>node</i>
+  /** Check if the tree contains node <i>node</i>
    * 
    * @param node
-   *          a data node
+   *          a node
    * @return true if <i>node</i> is already in the tree; false otherwise
    */
-  public boolean contains(DatanodeDescriptor node) {
+  public boolean contains(Node node) {
     if (node == null) return false;
     netlock.readLock().lock();
     try {
@@ -399,7 +407,7 @@
     }
   }
     
-  /** Return the total number of data nodes */
+  /** Return the total number of nodes */
   public int getNumOfLeaves() {
     netlock.readLock().lock();
     try {
@@ -409,16 +417,16 @@
     }
   }
     
-  /** Return the distance between two data nodes
+  /** Return the distance between two nodes
    * It is assumed that the distance from one node to its parent is 1
    * The distance between two nodes is calculated by summing up their distances
    * to their closest common  ancestor.
-   * @param node1 one data node
-   * @param node2 another data node
+   * @param node1 one node
+   * @param node2 another node
    * @return the distance between node1 and node2
    * node1 or node2 do not belong to the cluster
    */
-  public int getDistance(DatanodeDescriptor node1, DatanodeDescriptor node2) {
+  public int getDistance(Node node1, Node node2) {
     if (node1 == node2) {
       return 0;
     }
@@ -446,25 +454,24 @@
       netlock.readLock().unlock();
     }
     if (n1==null) {
-      LOG.warn("The cluster does not contain data node: "+node1.getPath());
+      LOG.warn("The cluster does not contain node: "+NodeBase.getPath(node1));
       return Integer.MAX_VALUE;
     }
     if (n2==null) {
-      LOG.warn("The cluster does not contain data node: "+node2.getPath());
+      LOG.warn("The cluster does not contain node: "+NodeBase.getPath(node2));
       return Integer.MAX_VALUE;
     }
     return dis+2;
   } 
     
-  /** Check if two data nodes are on the same rack
-   * @param node1 one data node
-   * @param node2 another data node
+  /** Check if two nodes are on the same rack
+   * @param node1 one node
+   * @param node2 another node
    * @return true if node1 and node2 are pm the same rack; false otherwise
    * @exception IllegalArgumentException when either node1 or node2 is null, or
    * node1 or node2 do not belong to the cluster
    */
-  public boolean isOnSameRack(
-                              DatanodeDescriptor node1, DatanodeDescriptor node2) {
+  public boolean isOnSameRack( Node node1,  Node node2) {
     if (node1 == null || node2 == null) {
       return false;
     }
@@ -479,12 +486,12 @@
     
   final private static Random r = new Random();
   /** randomly choose one node from <i>scope</i>
-   * if scope starts with ~, choose one from the all datanodes except for the
+   * if scope starts with ~, choose one from the all nodes except for the
    * ones in <i>scope</i>; otherwise, choose one from <i>scope</i>
-   * @param scope range of datanodes from which a node will be choosen
-   * @return the choosen data node
+   * @param scope range of nodes from which a node will be choosen
+   * @return the choosen node
    */
-  public DatanodeDescriptor chooseRandom(String scope) {
+  public Node chooseRandom(String scope) {
     netlock.readLock().lock();
     try {
       if (scope.startsWith("~")) {
@@ -497,7 +504,7 @@
     }
   }
     
-  private DatanodeDescriptor chooseRandom(String scope, String excludedScope){
+  private Node chooseRandom(String scope, String excludedScope){
     if (excludedScope != null) {
       if (scope.startsWith(excludedScope)) {
         return null;
@@ -507,8 +514,8 @@
       }
     }
     Node node = getNode(scope);
-    if (node instanceof DatanodeDescriptor) {
-      return (DatanodeDescriptor)node;
+    if (!(node instanceof InnerNode)) {
+      return node;
     }
     InnerNode innerNode = (InnerNode)node;
     int numOfDatanodes = innerNode.getNumOfLeaves();
@@ -516,7 +523,7 @@
       node = null;
     } else {
       node = getNode(excludedScope);
-      if (node instanceof DatanodeDescriptor) {
+      if (!(node instanceof InnerNode)) {
         numOfDatanodes -= 1;
       } else {
         numOfDatanodes -= ((InnerNode)node).getNumOfLeaves();
@@ -527,14 +534,14 @@
   }
        
   /** return the number of leaves in <i>scope</i> but not in <i>excludedNodes</i>
-   * if scope starts with ~, return the number of datanodes that are not
+   * if scope starts with ~, return the number of nodes that are not
    * in <i>scope</i> and <i>excludedNodes</i>; 
    * @param scope a path string that may start with ~
-   * @param excludedNodes a list of data nodes
-   * @return number of available data nodes
+   * @param excludedNodes a list of nodes
+   * @return number of available nodes
    */
   public int countNumOfAvailableNodes(String scope,
-                                      List<DatanodeDescriptor> excludedNodes) {
+                                      List<Node> excludedNodes) {
     boolean isExcluded=false;
     if (scope.startsWith("~")) {
       isExcluded=true;
@@ -544,8 +551,8 @@
     int count=0; // the number of nodes in both scope & excludedNodes
     netlock.readLock().lock();
     try {
-      for(DatanodeDescriptor node:excludedNodes) {
-        if ((node.getPath()+NodeBase.PATH_SEPARATOR_STR).
+      for(Node node:excludedNodes) {
+        if ((NodeBase.getPath(node)+NodeBase.PATH_SEPARATOR_STR).
             startsWith(scope+NodeBase.PATH_SEPARATOR_STR)) {
           count++;
         }
@@ -578,17 +585,17 @@
     tree.append("Expected number of leaves:");
     tree.append(numOfLeaves);
     tree.append("\n");
-    // print datanodes
+    // print nodes
     for(int i=0; i<numOfLeaves; i++) {
-      tree.append(clusterMap.getLeaf(i, null).getPath());
+      tree.append(NodeBase.getPath(clusterMap.getLeaf(i, null)));
       tree.append("\n");
     }
     return tree.toString();
   }
 
   /* swap two array items */
-  static private void swap(DatanodeDescriptor[] nodes, int i, int j) {
-    DatanodeDescriptor tempNode;
+  static private void swap(Node[] nodes, int i, int j) {
+    Node tempNode;
     tempNode = nodes[j];
     nodes[j] = nodes[i];
     nodes[i] = tempNode;
@@ -604,8 +611,7 @@
    * location at postion 0.
    * It leaves the rest nodes untouched.
    */
-  public synchronized void pseudoSortByDistance(
-      DatanodeDescriptor reader, DatanodeDescriptor[] nodes ) {
+  public synchronized void pseudoSortByDistance( Node reader, Node[] nodes ) {
     int tempIndex = 0;
     if (reader != null ) {
       int localRackNode = -1;

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/net/Node.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/net/Node.java?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/net/Node.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/net/Node.java Mon Sep 10 11:46:43 2007
@@ -34,8 +34,12 @@
   public String getName();
   /** Return this node's parent */
   public Node getParent();
+  /** Set this node's parent */
+  public void setParent(Node parent);
   /** Return this node's level in the tree.
    * E.g. the root of a tree returns 0 and its children return 1
    */
   public int getLevel();
+  /** Set this node's level in the tree.*/
+  public void setLevel(int i);
 }

Modified: lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NodeBase.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NodeBase.java?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NodeBase.java (original)
+++ lucene/hadoop/trunk/src/java/org/apache/hadoop/net/NodeBase.java Mon Sep 10 11:46:43 2007
@@ -85,13 +85,13 @@
   public String getNetworkLocation() { return location; }
   
   /** Return this node's path */
-  public String getPath() {
-    return location+PATH_SEPARATOR_STR+name;
+  public static String getPath(Node node) {
+    return node.getNetworkLocation()+PATH_SEPARATOR_STR+node.getName();
   }
   
   /** Return this node's string representation */
   public String toString() {
-    return getPath();
+    return getPath(this);
   }
 
   /** Normalize a path */
@@ -114,8 +114,18 @@
   /** Return this node's parent */
   public Node getParent() { return parent; }
   
+  /** Set this node's parent */
+  public void setParent(Node parent) {
+    this.parent = parent;
+  }
+  
   /** Return this node's level in the tree.
    * E.g. the root of a tree returns 0 and its children return 1
    */
   public int getLevel() { return level; }
+  
+  /** Set this node's level in the tree */
+  public void setLevel(int level) {
+    this.level = level;
+  }
 }

Modified: lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestReplicationPolicy.java
URL: http://svn.apache.org/viewvc/lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestReplicationPolicy.java?rev=574331&r1=574330&r2=574331&view=diff
==============================================================================
--- lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestReplicationPolicy.java (original)
+++ lucene/hadoop/trunk/src/test/org/apache/hadoop/dfs/TestReplicationPolicy.java Mon Sep 10 11:46:43 2007
@@ -24,6 +24,7 @@
 
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.net.NetworkTopology;
+import org.apache.hadoop.net.Node;
 
 import junit.framework.TestCase;
 
@@ -128,23 +129,23 @@
    * @throws Exception
    */
   public void testChooseTarget2() throws Exception { 
-    List<DatanodeDescriptor> excludedNodes;
+    List<Node> excludedNodes;
     DatanodeDescriptor[] targets;
     
-    excludedNodes = new ArrayList<DatanodeDescriptor>();
+    excludedNodes = new ArrayList<Node>();
     excludedNodes.add(dataNodes[1]); 
     targets = replicator.chooseTarget(
                                       0, dataNodes[0], excludedNodes, BLOCK_SIZE);
     assertEquals(targets.length, 0);
     
-    excludedNodes = new ArrayList<DatanodeDescriptor>();
+    excludedNodes.clear();
     excludedNodes.add(dataNodes[1]); 
     targets = replicator.chooseTarget(
                                       1, dataNodes[0], excludedNodes, BLOCK_SIZE);
     assertEquals(targets.length, 1);
     assertEquals(targets[0], dataNodes[0]);
     
-    excludedNodes = new ArrayList<DatanodeDescriptor>();
+    excludedNodes.clear();
     excludedNodes.add(dataNodes[1]); 
     targets = replicator.chooseTarget(
                                       2, dataNodes[0], excludedNodes, BLOCK_SIZE);
@@ -152,7 +153,7 @@
     assertEquals(targets[0], dataNodes[0]);
     assertFalse(cluster.isOnSameRack(targets[0], targets[1]));
     
-    excludedNodes = new ArrayList<DatanodeDescriptor>();
+    excludedNodes.clear();
     excludedNodes.add(dataNodes[1]); 
     targets = replicator.chooseTarget(
                                       3, dataNodes[0], excludedNodes, BLOCK_SIZE);
@@ -161,7 +162,7 @@
     assertFalse(cluster.isOnSameRack(targets[0], targets[1]));
     assertTrue(cluster.isOnSameRack(targets[1], targets[2]));
     
-    excludedNodes = new ArrayList<DatanodeDescriptor>();
+    excludedNodes.clear();
     excludedNodes.add(dataNodes[1]); 
     targets = replicator.chooseTarget(
                                       4, dataNodes[0], excludedNodes, BLOCK_SIZE);