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 br...@apache.org on 2021/07/30 03:13:08 UTC

[hadoop] 02/05: HADOOP-12430. Fix HDFS client gets errors trying to to connect to IPv6 DataNode. Contributed by Nate Edel.

This is an automated email from the ASF dual-hosted git repository.

brahma pushed a commit to branch HADOOP-17800
in repository https://gitbox.apache.org/repos/asf/hadoop.git

commit ddecfe1524b72e35c5483133d6e60ffc7418927c
Author: Brahma Reddy Battula <br...@apache.org>
AuthorDate: Mon Jul 26 17:18:55 2021 +0530

    HADOOP-12430. Fix HDFS client gets errors trying to to connect to IPv6 DataNode. Contributed by Nate Edel.
---
 .../main/java/org/apache/hadoop/net/NetUtils.java  | 160 +++++++++++++++++++--
 .../java/org/apache/hadoop/net/TestNetUtils.java   |   8 +-
 .../apache/hadoop/hdfs/protocol/DatanodeID.java    |  14 +-
 .../datatransfer/sasl/DataTransferSaslUtil.java    |   9 +-
 4 files changed, 162 insertions(+), 29 deletions(-)

diff --git a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/net/NetUtils.java b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/net/NetUtils.java
index 0f4dd9d..49fa540 100644
--- a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/net/NetUtils.java
+++ b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/net/NetUtils.java
@@ -40,7 +40,6 @@ import java.nio.channels.SocketChannel;
 import java.nio.channels.UnresolvedAddressException;
 import java.util.Map.Entry;
 import java.util.concurrent.TimeUnit;
-import java.util.regex.Pattern;
 import java.util.*;
 import java.util.concurrent.ConcurrentHashMap;
 
@@ -61,6 +60,11 @@ import org.apache.hadoop.ipc.VersionedProtocol;
 import org.apache.hadoop.security.SecurityUtil;
 import org.apache.hadoop.util.ReflectionUtils;
 
+import com.google.common.net.HostAndPort;
+import com.google.common.net.InetAddresses;
+import org.apache.http.conn.util.InetAddressUtils;
+import java.net.*;
+
 import org.apache.hadoop.thirdparty.com.google.common.base.Preconditions;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
@@ -70,7 +74,7 @@ import org.slf4j.LoggerFactory;
 public class NetUtils {
   private static final Logger LOG = LoggerFactory.getLogger(NetUtils.class);
   
-  private static Map<String, String> hostToResolved = 
+  private static Map<String, String> hostToResolved =
                                      new HashMap<String, String>();
   /** text to point users elsewhere: {@value} */
   private static final String FOR_MORE_DETAILS_SEE
@@ -669,9 +673,6 @@ public class NetUtils {
     }
   }
 
-  private static final Pattern ipPortPattern = // Pattern for matching ip[:port]
-    Pattern.compile("\\d{1,3}\\.\\d{1,3}\\.\\d{1,3}\\.\\d{1,3}(:\\d+)?");
-  
   /**
    * Attempt to obtain the host name of the given string which contains
    * an IP address and an optional port.
@@ -680,16 +681,26 @@ public class NetUtils {
    * @return Host name or null if the name can not be determined
    */
   public static String getHostNameOfIP(String ipPort) {
-    if (null == ipPort || !ipPortPattern.matcher(ipPort).matches()) {
+    String ip = null;
+    if (null == ipPort || ipPort.isEmpty()) {
       return null;
     }
-    
     try {
-      int colonIdx = ipPort.indexOf(':');
-      String ip = (-1 == colonIdx) ? ipPort
-          : ipPort.substring(0, ipPort.indexOf(':'));
+      HostAndPort hostAndPort = HostAndPort.fromString(ipPort);
+      ip = hostAndPort.getHost();
+      if (!InetAddresses.isInetAddress(ip)) {
+        return null;
+      }
+    } catch (IllegalArgumentException e) {
+      LOG.debug("getHostNameOfIP: '" + ipPort
+              + "' is not a valid IP address or IP/Port pair.", e);
+      return null;
+    }
+
+    try {
       return InetAddress.getByName(ip).getHostName();
     } catch (UnknownHostException e) {
+      LOG.trace("getHostNameOfIP: '"+ipPort+"' name not resolved.", e);
       return null;
     }
   }
@@ -702,8 +713,20 @@ public class NetUtils {
    * @return host:port
    */
   public static String normalizeIP2HostName(String ipPort) {
-    if (null == ipPort || !ipPortPattern.matcher(ipPort).matches()) {
-      return ipPort;
+    String ip = null;
+    if (null == ipPort || ipPort.isEmpty()) {
+      return null;
+    }
+    try {
+      HostAndPort hostAndPort = HostAndPort.fromString(ipPort);
+      ip = hostAndPort.getHost();
+      if (!InetAddresses.isInetAddress(ip)) {
+        return null;
+      }
+    } catch (IllegalArgumentException e) {
+      LOG.debug("getHostNameOfIP: '" + ipPort
+          + "' is not a valid IP address or IP/Port pair.", e);
+      return null;
     }
 
     InetSocketAddress address = createSocketAddr(ipPort);
@@ -735,11 +758,88 @@ public class NetUtils {
 
   /**
    * Compose a "host:port" string from the address.
+   *
+   * Note that this preferentially returns the host name if available; if the
+   * IP address is desired, use getIPPortString(); if both are desired as in
+   * InetSocketAddress.toString, use getSocketAddressString()
    */
   public static String getHostPortString(InetSocketAddress addr) {
-    return addr.getHostName() + ":" + addr.getPort();
+    String hostName = addr.getHostName();
+    if (InetAddressUtils.isIPv6Address(hostName)) {
+      return "[" + hostName + "]:" + addr.getPort();
+    }
+    return hostName + ":" + addr.getPort();
   }
-  
+  /**
+   * Compose a "ip:port" string from the InetSocketAddress.
+   *
+   * Note that this may result in an NPE if passed an unresolved
+   * InetSocketAddress.
+   */
+  public static String getIPPortString(InetSocketAddress addr) {
+    final InetAddress ip = addr.getAddress();
+    // this is a judgement call, and we might arguably just guard against NPE
+    // by treating null as "" ; I think this is going to hide more bugs than it
+    // prevents
+    if (ip == null) {
+      throw new IllegalArgumentException(
+              "getIPPortString called with unresolved InetSocketAddress : "
+                      + getSocketAddressString(addr));
+    }
+    String ipString = ip.getHostAddress();
+    if (ip instanceof Inet6Address) {
+      return "[" + ipString + "]:" + addr.getPort();
+    }
+    return ipString + ":" + addr.getPort();
+  }
+
+  public static String getIPPortString(String ipAddr, int port) {
+    String s;
+    if (ipAddr != null) {
+      s = ipAddr + ":" + port;
+    } else {
+      s = ":" + port;
+    }
+    //Blank eventually will get to treated as localhost if this gets down to
+    // InetAddress. Tests extensively use a blank address, and we don't want
+    // to change behavior here.
+    if (ipAddr != null && !ipAddr.isEmpty() && InetAddressUtils
+        .isIPv6Address(ipAddr)) {
+      try {
+        InetAddress addr = InetAddress.getByName(ipAddr);
+        String cleanAddr = addr.getHostAddress();
+        if (addr instanceof Inet6Address) {
+          s = '[' + cleanAddr + ']' + ":" + port;
+        }
+      } catch (UnknownHostException e) {
+        // ignore anything that isn't an IPv6 literal and keep the old
+        // behavior. could add debug log here, but this should only happen
+        // if there's a bug in InetAddressUtils.isIPv6Address which accepts
+        // something that isn't an IPv6 literal.
+      }
+    }
+    return s;
+  }
+
+  /**
+   * An IPv6-safe version of InetSocketAddress.toString().
+   * Note that this will typically be of the form hostname/IP:port and is NOT
+   * a substitute for getHostPortString or getIPPortString.
+   */
+  public static String getSocketAddressString(InetSocketAddress addr) {
+    if (addr.isUnresolved()) {
+      return addr.toString();
+    }
+    InetAddress ip = addr.getAddress();
+    if (ip instanceof Inet6Address) {
+      String hostName = addr.getHostName();
+      return ((hostName != null) ? hostName : "")
+              + "/[" + ip.getHostAddress() + "]:" + addr.getPort();
+    } else {
+      return addr.toString();
+    }
+  }
+
   /**
    * Checks if {@code host} is a local host name and return {@link InetAddress}
    * corresponding to that address.
@@ -1037,6 +1137,38 @@ public class NetUtils {
   }
 
   /**
+   * Wrapper method on HostAndPort; returns the port from a host:port
+   * or IP:port pair.
+   *
+   * It's probably best to create your own HostAndPort.fromString(hp) and
+   * do a .getPort and .getHostText if you need both host and port in one
+   * scope.
+   */
+  public static int getPortFromHostPort(String hp) {
+    return HostAndPort.fromString(hp).getPort();
+  }
+
+  /**
+   * Wrapper method on HostAndPort; returns the host from a host:port
+   * or IP:port pair.
+   *
+   * It's probably best to create your own HostAndPort.fromString(hp) and
+   * do a .getPort and .getHostText if you need both host and port in one
+   * scope.
+   */
+  public static String getHostFromHostPort(String hp) {
+    return HostAndPort.fromString(hp).getHost();
+  }
+
+  public static InetAddress getInetAddressFromInetSocketAddressString(
+          String remoteAddr) {
+    int slashIdx = remoteAddr.indexOf('/') + 1;
+    int colonIdx = remoteAddr.lastIndexOf(':');
+    String ipOnly = remoteAddr.substring(slashIdx, colonIdx);
+    return InetAddresses.forString(ipOnly);
+  }
+
+  /**
    * Return an @{@link InetAddress} to bind to. If bindWildCardAddress is true
    * than returns null.
    *
diff --git a/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/net/TestNetUtils.java b/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/net/TestNetUtils.java
index cfffd85..9840442 100644
--- a/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/net/TestNetUtils.java
+++ b/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/net/TestNetUtils.java
@@ -721,8 +721,8 @@ public class TestNetUtils {
     } catch (UnknownHostException e) {
       Assume.assumeTrue("Network not resolving "+ oneHost, false);
     }
-    List<String> hosts = Arrays.asList("127.0.0.1",
-        "localhost", oneHost, "UnknownHost123");
+    List<String> hosts = Arrays.asList(new String[] {"127.0.0.1",
+        "localhost", oneHost, "UnknownHost123.invalid"});
     List<String> normalizedHosts = NetUtils.normalizeHostNames(hosts);
     String summary = "original [" + StringUtils.join(hosts, ", ") + "]"
         + " normalized [" + StringUtils.join(normalizedHosts, ", ") + "]";
@@ -745,11 +745,13 @@ public class TestNetUtils {
     assertNull(NetUtils.getHostNameOfIP(null));
     assertNull(NetUtils.getHostNameOfIP(""));
     assertNull(NetUtils.getHostNameOfIP("crazytown"));
-    assertNull(NetUtils.getHostNameOfIP("127.0.0.1:"));   // no port
     assertNull(NetUtils.getHostNameOfIP("127.0.0.1:-1")); // bogus port
     assertNull(NetUtils.getHostNameOfIP("127.0.0.1:A"));  // bogus port
+    assertNotNull(NetUtils.getHostNameOfIP("[::1]"));
+    assertNotNull(NetUtils.getHostNameOfIP("[::1]:1"));
     assertNotNull(NetUtils.getHostNameOfIP("127.0.0.1"));
     assertNotNull(NetUtils.getHostNameOfIP("127.0.0.1:1"));
+    assertEquals("localhost", NetUtils.getHostNameOfIP("127.0.0.1:"));
   }
 
   @Test
diff --git a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/DatanodeID.java b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/DatanodeID.java
index 2cb1687..ded7c9a 100644
--- a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/DatanodeID.java
+++ b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/DatanodeID.java
@@ -23,7 +23,7 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 import org.apache.hadoop.thirdparty.com.google.common.annotations.VisibleForTesting;
-
+import org.apache.hadoop.net.NetUtils;
 import java.net.InetSocketAddress;
 
 /**
@@ -125,8 +125,9 @@ public class DatanodeID implements Comparable<DatanodeID> {
   }
 
   public void setIpAddr(String ipAddr) {
+    this.ipAddr = ipAddr;
     //updated during registration, preserve former xferPort
-    setIpAndXferPort(ipAddr, getByteString(ipAddr), xferPort);
+    setIpAndXferPort(this.ipAddr, getByteString(ipAddr), xferPort);
   }
 
   private void setIpAndXferPort(String ipAddr, ByteString ipAddrBytes,
@@ -135,7 +136,7 @@ public class DatanodeID implements Comparable<DatanodeID> {
     this.ipAddr = ipAddr;
     this.ipAddrBytes = ipAddrBytes;
     this.xferPort = xferPort;
-    this.xferAddr = ipAddr + ":" + xferPort;
+    this.xferAddr = NetUtils.getIPPortString(ipAddr, xferPort);
   }
 
   public void setPeerHostName(String peerHostName) {
@@ -201,21 +202,21 @@ public class DatanodeID implements Comparable<DatanodeID> {
    * @return IP:ipcPort string
    */
   private String getIpcAddr() {
-    return ipAddr + ":" + ipcPort;
+    return NetUtils.getIPPortString(ipAddr, ipcPort);
   }
 
   /**
    * @return IP:infoPort string
    */
   public String getInfoAddr() {
-    return ipAddr + ":" + infoPort;
+    return NetUtils.getIPPortString(ipAddr, infoPort);
   }
 
   /**
    * @return IP:infoPort string
    */
   public String getInfoSecureAddr() {
-    return ipAddr + ":" + infoSecurePort;
+    return NetUtils.getIPPortString(ipAddr, infoSecurePort);
   }
 
   /**
@@ -299,6 +300,7 @@ public class DatanodeID implements Comparable<DatanodeID> {
    * Note that this does not update storageID.
    */
   public void updateRegInfo(DatanodeID nodeReg) {
+    ipAddr = nodeReg.getIpAddr();
     setIpAndXferPort(nodeReg.getIpAddr(), nodeReg.getIpAddrBytes(),
         nodeReg.getXferPort());
     hostName = nodeReg.getHostName();
diff --git a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/datatransfer/sasl/DataTransferSaslUtil.java b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/datatransfer/sasl/DataTransferSaslUtil.java
index 526f3d0..33f1bfe 100644
--- a/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/datatransfer/sasl/DataTransferSaslUtil.java
+++ b/hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/hdfs/protocol/datatransfer/sasl/DataTransferSaslUtil.java
@@ -52,6 +52,7 @@ import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.DataTransferEncr
 import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.HandshakeSecretProto;
 import org.apache.hadoop.hdfs.protocol.proto.HdfsProtos.CipherOptionProto;
 import org.apache.hadoop.hdfs.protocolPB.PBHelperClient;
+import org.apache.hadoop.net.NetUtils;
 import org.apache.hadoop.security.SaslPropertiesResolver;
 import org.apache.hadoop.security.SaslRpcServer.QualityOfProtection;
 import org.slf4j.Logger;
@@ -60,7 +61,6 @@ import org.slf4j.LoggerFactory;
 import org.apache.hadoop.thirdparty.com.google.common.base.Charsets;
 import org.apache.hadoop.thirdparty.com.google.common.collect.ImmutableSet;
 import org.apache.hadoop.thirdparty.com.google.common.collect.Maps;
-import org.apache.hadoop.thirdparty.com.google.common.net.InetAddresses;
 import org.apache.hadoop.thirdparty.protobuf.ByteString;
 
 /**
@@ -157,11 +157,8 @@ public final class DataTransferSaslUtil {
    * @return InetAddress from peer
    */
   public static InetAddress getPeerAddress(Peer peer) {
-    String remoteAddr = peer.getRemoteAddressString().split(":")[0];
-    int slashIdx = remoteAddr.indexOf('/');
-    return InetAddresses.forString(slashIdx != -1 ?
-        remoteAddr.substring(slashIdx + 1, remoteAddr.length()) :
-        remoteAddr);
+    String remoteAddr = peer.getRemoteAddressString();
+    return NetUtils.getInetAddressFromInetSocketAddressString(remoteAddr);
   }
 
   /**

---------------------------------------------------------------------
To unsubscribe, e-mail: common-commits-unsubscribe@hadoop.apache.org
For additional commands, e-mail: common-commits-help@hadoop.apache.org