소스 검색

HDFS-12697. Ozone services must stay disabled in secure setup for alpha. Contributed by Bharat Viswanadham.

Xiaoyu Yao 7 년 전
부모
커밋
be6bf1b80f

+ 8 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/bin/start-ozone.sh

@@ -44,6 +44,14 @@ else
   exit 1
 fi
 
+SECURITY_ENABLED=$("${HADOOP_HDFS_HOME}/bin/hdfs" getconf -confKey hadoop.security.authentication | tr '[:upper:]' '[:lower:]' 2>&-)
+SECURITY_AUTHORIZATION_ENABLED=$("${HADOOP_HDFS_HOME}/bin/hdfs" getconf -confKey hadoop.security.authorization | tr '[:upper:]' '[:lower:]' 2>&-)
+
+if [[ ${SECURITY_ENABLED} == "kerberos" || ${SECURITY_AUTHORIZATION_ENABLED} == "true" ]]; then
+  echo "Ozone is not supported in a security enabled cluster."
+  exit 1
+fi
+
 #---------------------------------------------------------
 # Check if ozone is enabled
 OZONE_ENABLED=$("${HADOOP_HDFS_HOME}/bin/hdfs" getconf -confKey ozone.enabled | tr '[:upper:]' '[:lower:]' 2>&-)

+ 8 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/bin/stop-ozone.sh

@@ -44,6 +44,14 @@ else
   exit 1
 fi
 
+SECURITY_ENABLED=$("${HADOOP_HDFS_HOME}/bin/hdfs" getconf -confKey hadoop.security.authentication | tr '[:upper:]' '[:lower:]' 2>&-)
+SECURITY_AUTHORIZATION_ENABLED=$("${HADOOP_HDFS_HOME}/bin/hdfs" getconf -confKey hadoop.security.authorization | tr '[:upper:]' '[:lower:]' 2>&-)
+
+if [[ ${SECURITY_ENABLED} == "kerberos" || ${SECURITY_AUTHORIZATION_ENABLED} == "true" ]]; then
+  echo "Ozone is not supported in a security enabled cluster."
+  exit 1
+fi
+
 #---------------------------------------------------------
 # Check if ozone is enabled
 OZONE_ENABLED=$("${HADOOP_HDFS_HOME}/bin/hdfs" getconf -confKey ozone.enabled | tr '[:upper:]' '[:lower:]' 2>&-)

+ 20 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSUtil.java

@@ -36,6 +36,8 @@ import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMESERVICE_ID;
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_SERVER_HTTPS_KEYPASSWORD_KEY;
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_SERVER_HTTPS_KEYSTORE_PASSWORD_KEY;
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_SERVER_HTTPS_TRUSTSTORE_PASSWORD_KEY;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_ENABLED;
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_ENABLED_DEFAULT;
 
 import java.io.ByteArrayInputStream;
 import java.io.DataInputStream;
@@ -71,6 +73,7 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.crypto.key.KeyProvider;
 import org.apache.hadoop.crypto.key.KeyProviderCryptoExtension;
+import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
@@ -1526,6 +1529,23 @@ public class DFSUtil {
     return password;
   }
 
+  public static boolean isOzoneEnabled(Configuration conf) {
+    String securityEnabled = conf.get(CommonConfigurationKeysPublic
+            .HADOOP_SECURITY_AUTHENTICATION,
+        "simple");
+    boolean securityAuthorizationEnabled = conf.getBoolean(
+        CommonConfigurationKeysPublic.HADOOP_SECURITY_AUTHORIZATION,
+        false);
+
+    if (securityEnabled.equals("kerberos") || securityAuthorizationEnabled) {
+      LOG.error("Ozone is not supported in a security enabled cluster. ");
+      return false;
+    } else {
+      return conf.getBoolean(OZONE_ENABLED,
+          OZONE_ENABLED_DEFAULT);
+    }
+  }
+
   /**
    * Converts a Date into an ISO-8601 formatted datetime string.
    */

+ 2 - 4
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/DataNode.java

@@ -46,8 +46,6 @@ import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_MAX_NUM_BLOCKS_TO_LOG_DEF
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_MAX_NUM_BLOCKS_TO_LOG_KEY;
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_DATANODE_METRICS_LOGGER_PERIOD_SECONDS_DEFAULT;
 import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_DATANODE_METRICS_LOGGER_PERIOD_SECONDS_KEY;
-import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_ENABLED_DEFAULT;
-import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_ENABLED;
 import static org.apache.hadoop.util.ExitUtil.terminate;
 
 import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
@@ -475,8 +473,8 @@ public class DataNode extends ReconfigurableBase
     this.pipelineSupportECN = conf.getBoolean(
         DFSConfigKeys.DFS_PIPELINE_ECN_ENABLED,
         DFSConfigKeys.DFS_PIPELINE_ECN_ENABLED_DEFAULT);
-    this.ozoneEnabled = conf.getBoolean(OZONE_ENABLED,
-        OZONE_ENABLED_DEFAULT);
+
+    this.ozoneEnabled = DFSUtil.isOzoneEnabled(conf);
 
     confVersion = "core-" +
         conf.get("hadoop.common.configuration.version", "UNSPECIFIED") +

+ 24 - 9
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/web/URLDispatcher.java

@@ -68,33 +68,48 @@ class URLDispatcher extends SimpleChannelInboundHandler<HttpRequest> {
       p.replace(this,
           RequestDispatchObjectStoreChannelHandler.class.getSimpleName(), h);
       h.channelRead0(ctx, req);
-    } else {
+    } else if (!isObjectStoreRequestHeaders(req)){
       SimpleHttpProxyHandler h = new SimpleHttpProxyHandler(proxyHost);
       p.replace(this, SimpleHttpProxyHandler.class.getSimpleName(), h);
       h.channelRead0(ctx, req);
     }
   }
 
+
   /*
-   * Returns true if the request is to be handled by the object store.
+   * Returns true if the request has ozone headers
    *
    * @param req HTTP request
-   * @return true if the request is to be handled by the object store
+   * @return true if request has ozone header, else false
    */
-  private boolean isObjectStoreRequest(HttpRequest req) {
-    if (this.objectStoreJerseyContainer == null) {
-      LOG.debug("ozone : dispatching call to webHDFS");
-      return false;
-    }
+
+  private boolean isObjectStoreRequestHeaders(HttpRequest req) {
     for (String version : req.headers().getAll(Header.OZONE_VERSION_HEADER)) {
       if (version != null) {
-        LOG.debug("ozone : dispatching call to Ozone");
+        LOG.debug("ozone : dispatching call to Ozone, when security is not " +
+            "enabled");
         return true;
       }
     }
     return false;
   }
 
+
+  /*
+   * Returns true if the request is to be handled by the object store.
+   *
+   * @param req HTTP request
+   * @return true if the request is to be handled by the object store
+   */
+  private boolean isObjectStoreRequest(HttpRequest req) {
+    if (this.objectStoreJerseyContainer == null) {
+      LOG.debug("ozone : ozone is disabled or when security is enabled, ozone" +
+          " is not supported");
+      return false;
+    }
+    return isObjectStoreRequestHeaders(req);
+  }
+
   /**
    * Returns true if the request is to be handled by WebHDFS.
    *

+ 8 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/ksm/KeySpaceManager.java

@@ -57,6 +57,7 @@ import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
 
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_ENABLED;
 import static org.apache.hadoop.ozone.ksm.KSMConfigKeys
     .OZONE_KSM_ADDRESS_KEY;
 import static org.apache.hadoop.ozone.ksm.KSMConfigKeys
@@ -186,7 +187,13 @@ public class KeySpaceManager extends ServiceRuntimeInfoImpl
   public static void main(String[] argv) throws IOException {
     StringUtils.startupShutdownMessage(KeySpaceManager.class, argv, LOG);
     try {
-      KeySpaceManager ksm = new KeySpaceManager(new OzoneConfiguration());
+      OzoneConfiguration conf = new OzoneConfiguration();
+      if (!DFSUtil.isOzoneEnabled(conf)) {
+        System.out.println("KSM cannot be started in secure mode or when " +
+            OZONE_ENABLED + " is set to false");
+        System.exit(1);
+      }
+      KeySpaceManager ksm = new KeySpaceManager(conf);
       ksm.start();
       ksm.join();
     } catch (Throwable t) {

+ 8 - 2
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/StorageContainerManager.java

@@ -104,6 +104,7 @@ import java.util.UUID;
 import java.util.Collections;
 import java.util.stream.Collectors;
 
+import static org.apache.hadoop.ozone.OzoneConfigKeys.OZONE_ENABLED;
 import static org.apache.hadoop.ozone.protocol.proto
     .ScmBlockLocationProtocolProtos.DeleteScmBlockResult.Result;
 import static org.apache.hadoop.scm.ScmConfigKeys
@@ -321,8 +322,13 @@ public class StorageContainerManager extends ServiceRuntimeInfoImpl
     StringUtils.startupShutdownMessage(StorageContainerManager.class,
         argv, LOG);
     try {
-      StorageContainerManager scm = new StorageContainerManager(
-          new OzoneConfiguration());
+      OzoneConfiguration conf = new OzoneConfiguration();
+      if (!DFSUtil.isOzoneEnabled(conf)) {
+        System.out.println("SCM cannot be started in secure mode or when " +
+            OZONE_ENABLED + " is set to false");
+        System.exit(1);
+      }
+      StorageContainerManager scm = new StorageContainerManager(conf);
       scm.start();
       scm.join();
     } catch (Throwable t) {