瀏覽代碼

HADOOP-7314. Add support for throwing UnknownHostException when a host doesn't resolve. Needed for MAPREDUCE-2489. Contributed by Jeffrey Naisbitt.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.20-security@1148583 13f79535-47bb-0310-9956-ffa450edef68
Matthew Foley 14 年之前
父節點
當前提交
f11ab1ac4e

+ 3 - 0
CHANGES.txt

@@ -23,6 +23,9 @@ Release 0.20.205.0 - unreleased
     HADOOP-7432. Back-port HADOOP-7110 to 0.20-security: Implement chmod
     in NativeIO library. (Sherry Chen via mattf)
 
+    HADOOP-7314. Add support for throwing UnknownHostException when a host
+    doesn't resolve. Needed for MAPREDUCE-2489. (Jeffrey Naisbitt via mattf)
+
 Release 0.20.204.0 - unreleased
 
   NEW FEATURES

+ 72 - 20
src/core/org/apache/hadoop/net/CachedDNSToSwitchMapping.java

@@ -17,6 +17,9 @@
  */
 package org.apache.hadoop.net;
 
+import java.net.InetAddress;
+import java.net.SocketException;
+import java.net.UnknownHostException;
 import java.util.ArrayList;
 import java.util.List;
 import java.util.Map;
@@ -37,16 +40,11 @@ public class CachedDNSToSwitchMapping implements DNSToSwitchMapping {
     this.rawMapping = rawMapping;
   }
   
-  public List<String> resolve(List<String> names) {
-    // normalize all input names to be in the form of IP addresses
-    names = NetUtils.normalizeHostNames(names);
-    
-    List <String> result = new ArrayList<String>(names.size());
-    if (names.isEmpty()) {
-      return result;
-    }
-
-
+  
+  /**
+   * Returns the hosts from 'names' that have not been cached previously
+   */
+  private List<String> getUncachedHosts(List<String> names) {
     // find out all names without cached resolved location
     List<String> unCachedHosts = new ArrayList<String>(names.size());
     for (String name : names) {
@@ -54,27 +52,81 @@ public class CachedDNSToSwitchMapping implements DNSToSwitchMapping {
         unCachedHosts.add(name);
       } 
     }
-    
-    // Resolve those names
-    List<String> rNames = rawMapping.resolve(unCachedHosts);
-    
+    return unCachedHosts;
+  }
+  
+  /**
+   * Caches the resolved hosts
+   */
+  private void cacheResolvedHosts(List<String> uncachedHosts, 
+      List<String> resolvedHosts) {
     // Cache the result
-    if (rNames != null) {
-      for (int i=0; i<unCachedHosts.size(); i++) {
-        cache.put(unCachedHosts.get(i), rNames.get(i));
+    if (resolvedHosts != null) {
+      for (int i=0; i<uncachedHosts.size(); i++) {
+        cache.put(uncachedHosts.get(i), resolvedHosts.get(i));
       }
     }
-    
+  }
+  
+  /**
+   * Returns the cached resolution of the list of hostnames/addresses.
+   * Returns null if any of the names are not currently in the cache
+   */
+  private List<String> getCachedHosts(List<String> names) {
+    List<String> result = new ArrayList<String>(names.size());
     // Construct the result
     for (String name : names) {
-      //now everything is in the cache
       String networkLocation = cache.get(name);
       if (networkLocation != null) {
         result.add(networkLocation);
-      } else { //resolve all or nothing
+      } else {
         return null;
       }
     }
     return result;
   }
+  
+  /**
+   * Resolves host names and adds them to the cache.
+   * Unlike the 'resolve" method, this won't hide UnknownHostExceptions
+   * 
+   * @param names to resolve
+   * @return List of resolved names
+   * @throws UnknownHostException if any hosts cannot be resolved
+   */  
+  public List<String> resolveValidHosts(List<String> names) 
+    throws UnknownHostException {
+    if (names.isEmpty()) {
+      return new ArrayList<String>();
+    }
+    List<String> addresses = new ArrayList<String>(names.size());
+    for (String name : names) {
+      addresses.add(InetAddress.getByName(name).getHostAddress());
+    }
+    
+    List<String> uncachedHosts = this.getUncachedHosts(names);
+
+    // Resolve the uncached hosts
+    List<String> resolvedHosts = rawMapping.resolveValidHosts(uncachedHosts);
+    this.cacheResolvedHosts(uncachedHosts, resolvedHosts);
+    return this.getCachedHosts(addresses);
+  }
+  
+  public List<String> resolve(List<String> names) {
+    // normalize all input names to be in the form of IP addresses
+    names = NetUtils.normalizeHostNames(names);
+
+    List <String> result = new ArrayList<String>(names.size());
+    if (names.isEmpty()) {
+      return result;
+    }
+
+    List<String> uncachedHosts = this.getUncachedHosts(names);
+
+    // Resolve the uncached hosts
+    List<String> resolvedHosts = rawMapping.resolve(uncachedHosts);
+    this.cacheResolvedHosts(uncachedHosts, resolvedHosts);
+    return this.getCachedHosts(names);
+
+  }
 }

+ 20 - 0
src/core/org/apache/hadoop/net/DNSToSwitchMapping.java

@@ -18,6 +18,7 @@
 package org.apache.hadoop.net;
 
 import java.util.List;
+import java.net.UnknownHostException;
 
 /**
  * An interface that should be implemented to allow pluggable 
@@ -39,4 +40,23 @@ public interface DNSToSwitchMapping {
    * @return list of resolved network paths
    */
   public List<String> resolve(List<String> names);
+
+  /**
+   * Resolves a list of DNS-names/IP-addresses and returns back a list of
+   * switch information (network paths). One-to-one correspondence must be 
+   * maintained between the elements in the lists. 
+   * Consider an element in the argument list - x.y.com. The switch information
+   * that is returned must be a network path of the form /foo/rack, 
+   * where / is the root, and 'foo' is the switch where 'rack' is connected.
+   * Note the hostname/ip-address is not part of the returned path.
+   * The network topology of the cluster would determine the number of
+   * components in the network path.  Unlike 'resolve', names must be 
+   * resolvable
+   * @param names
+   * @return list of resolved network paths
+   * @throws UnknownHostException if any hosts are not resolvable
+   */
+  public List<String> resolveValidHosts(List<String> names)
+    throws UnknownHostException;
+
 }

+ 12 - 0
src/core/org/apache/hadoop/net/ScriptBasedMapping.java

@@ -20,6 +20,7 @@ package org.apache.hadoop.net;
 
 import java.util.*;
 import java.io.*;
+import java.net.UnknownHostException;
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
@@ -115,6 +116,17 @@ implements Configurable
     return m;
   }
   
+  public List<String> resolveValidHosts(List<String> names) 
+    throws UnknownHostException {
+    List<String> result = this.resolve(names);
+    if (result != null) {
+      return result;
+    } else {
+      throw new UnknownHostException(
+          "Unknown host(s) returned from ScriptBasedMapping");
+    }
+  }
+  
   private String runResolveCommand(List<String> args) {
     int loopCount = 0;
     if (args.size() == 0) {

+ 16 - 0
src/test/org/apache/hadoop/net/StaticMapping.java

@@ -18,6 +18,7 @@
 package org.apache.hadoop.net;
 
 import java.util.*;
+import java.net.UnknownHostException;
 
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.conf.Configured;
@@ -59,4 +60,19 @@ public class StaticMapping extends Configured implements DNSToSwitchMapping {
       return m;
     }
   }
+  public List<String> resolveValidHosts(List<String> names)
+    throws UnknownHostException {
+    List<String> m = new ArrayList<String>();
+    synchronized (nameToRackMap) {
+      for (String name : names) {
+        String rackId;
+        if ((rackId = nameToRackMap.get(name)) != null) {
+          m.add(rackId);
+        } else {
+          throw new UnknownHostException(name);
+        }
+      }
+      return m;
+    }
+  }
 }

+ 46 - 7
src/test/org/apache/hadoop/net/TestScriptBasedMapping.java

@@ -19,28 +19,67 @@ package org.apache.hadoop.net;
 
 import java.util.ArrayList;
 import java.util.List;
+import java.net.UnknownHostException;
 
 import org.apache.hadoop.conf.Configuration;
 
 import junit.framework.TestCase;
 
 public class TestScriptBasedMapping extends TestCase {
+  
+  private ScriptBasedMapping mapping;
+  private Configuration conf;
+  private List<String> names;
+  
+  public TestScriptBasedMapping() {
+    mapping = new ScriptBasedMapping();
 
-  public void testNoArgsMeansNoResult() {
-    ScriptBasedMapping mapping = new ScriptBasedMapping();
-
-    Configuration conf = new Configuration();
+    conf = new Configuration();
     conf.setInt(ScriptBasedMapping.SCRIPT_ARG_COUNT_KEY,
         ScriptBasedMapping.MIN_ALLOWABLE_ARGS - 1);
     conf.set(ScriptBasedMapping.SCRIPT_FILENAME_KEY, "any-filename");
 
-    mapping.setConf(conf);
+    mapping.setConf(conf);    
+  }
 
-    List<String> names = new ArrayList<String>();
+  public void testNoArgsMeansNoResult() {
+    names = new ArrayList<String>();
     names.add("some.machine.name");
     names.add("other.machine.name");
-
     List<String> result = mapping.resolve(names);
     assertNull(result);
   }
+  
+  public void testResolveValidInvalidHostException() {
+    names = new ArrayList<String>();
+    names.add("1.com"); // Add invalid hostname that doesn't resolve
+    boolean exceptionThrown = false;
+    try {
+      mapping.resolveValidHosts(names);
+    } catch (UnknownHostException e) {
+      exceptionThrown = true;
+    }
+    assertTrue(
+        "resolveValidHosts did not throw UnknownHostException for invalid host",
+        exceptionThrown);
+  }
+
+  public void testResolveValidHostNoException() {
+    conf.setInt(ScriptBasedMapping.SCRIPT_ARG_COUNT_KEY,
+        ScriptBasedMapping.MIN_ALLOWABLE_ARGS);
+    conf.set(ScriptBasedMapping.SCRIPT_FILENAME_KEY, "echo");
+    mapping.setConf(conf);    
+
+    names = new ArrayList<String>();
+    names.add("some.machine.name");
+    names.add("other.machine.name");
+    
+    boolean exceptionThrown = false;
+    try {
+      mapping.resolveValidHosts(names);
+    } catch (UnknownHostException e) {
+      exceptionThrown = true;
+    }
+    assertFalse("resolveValidHosts threw Exception for valid host", exceptionThrown);
+  }
 }