Преглед на файлове

HDFS-12007. Ozone: Enable HttpServer2 for SCM and KSM. Contributed by Elek, Marton.

Anu Engineer преди 8 години
родител
ревизия
31eafb15bc
променени са 23 файла, в които са добавени 1211 реда и са изтрити 28 реда
  1. 17 0
      hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/scm/ScmConfigKeys.java
  2. 2 2
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/cblock/CBlockManager.java
  3. 23 7
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/OzoneClientUtils.java
  4. 3 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java
  5. 16 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/ksm/KSMConfigKeys.java
  6. 8 4
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/ksm/KeySpaceManager.java
  7. 74 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/ksm/KeySpaceManagerHttpServer.java
  8. 33 11
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/StorageContainerManager.java
  9. 77 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/StorageContainerManagerHttpServer.java
  10. 178 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/web/OzoneHttpServer.java
  11. 24 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/web/package-info.java
  12. 129 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/resources/ozone-default.xml
  13. 74 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/ksm/index.html
  14. 23 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/ksm/main.css
  15. 75 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/scm/index.html
  16. 23 0
      hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/scm/main.css
  17. 6 3
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNameNodeHttpServer.java
  18. 4 1
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/MiniOzoneCluster.java
  19. 141 0
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/ksm/TestKeySpaceManagerHttpServer.java
  20. 21 0
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/ksm/package-info.java
  21. 141 0
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/scm/TestStorageContainerManagerHttpServer.java
  22. 98 0
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/web/TestOzoneHttpServer.java
  23. 21 0
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/web/package-info.java

+ 17 - 0
hadoop-hdfs-project/hadoop-hdfs-client/src/main/java/org/apache/hadoop/scm/ScmConfigKeys.java

@@ -91,6 +91,23 @@ public final class ScmConfigKeys {
   public static final String OZONE_SCM_DATANODE_BIND_HOST_DEFAULT =
       "0.0.0.0";
 
+  public static final String OZONE_SCM_HTTP_ENABLED_KEY =
+      "ozone.scm.http.enabled";
+  public static final String OZONE_SCM_HTTP_BIND_HOST_KEY =
+      "ozone.scm.http-bind-host";
+  public static final String OZONE_SCM_HTTPS_BIND_HOST_KEY =
+      "ozone.scm.https-bind-host";
+  public static final String OZONE_SCM_HTTP_ADDRESS_KEY =
+      "ozone.scm.http-address";
+  public static final String OZONE_SCM_HTTPS_ADDRESS_KEY =
+      "ozone.scm.https-address";
+  public static final String OZONE_SCM_KEYTAB_FILE =
+      "ozone.scm.keytab.file";
+  public static final String OZONE_SCM_HTTP_BIND_HOST_DEFAULT = "0.0.0.0";
+  public static final int OZONE_SCM_HTTP_BIND_PORT_DEFAULT = 9876;
+  public static final int OZONE_SCM_HTTPS_BIND_PORT_DEFAULT = 9877;
+
+
   public static final String OZONE_SCM_HANDLER_COUNT_KEY =
       "ozone.scm.handler.count.key";
   public static final int OZONE_SCM_HANDLER_COUNT_DEFAULT = 10;

+ 2 - 2
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/cblock/CBlockManager.java

@@ -149,7 +149,7 @@ public class CBlockManager implements CBlockServiceProtocol,
         DFS_CBLOCK_SERVICERPC_HANDLER_COUNT_KEY,
         DFS_CBLOCK_SERVICERPC_HANDLER_COUNT_DEFAULT);
     InetSocketAddress cblockServiceRpcAddress =
-        OzoneClientUtils.updateListenAddress(conf,
+        OzoneClientUtils.updateRPCListenAddress(conf,
             DFS_CBLOCK_SERVICERPC_ADDRESS_KEY, serviceRpcAddr, cblockService);
     LOG.info("CBlock manager listening for client commands on: {}",
         cblockServiceRpcAddress);
@@ -170,7 +170,7 @@ public class CBlockManager implements CBlockServiceProtocol,
         DFS_CBLOCK_SERVICERPC_HANDLER_COUNT_KEY,
         DFS_CBLOCK_SERVICERPC_HANDLER_COUNT_DEFAULT);
     InetSocketAddress cblockServerRpcAddress =
-        OzoneClientUtils.updateListenAddress(conf,
+        OzoneClientUtils.updateRPCListenAddress(conf,
             DFS_CBLOCK_JSCSIRPC_ADDRESS_KEY, serverRpcAddr, cblockServer);
     LOG.info("CBlock server listening for client commands on: {}",
         cblockServerRpcAddress);

+ 23 - 7
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/OzoneClientUtils.java

@@ -361,8 +361,8 @@ public final class OzoneClientUtils {
    * @throws IllegalArgumentException if any values are not in the 'host'
    *             or host:port format.
    */
-  static Optional<String> getHostNameFromConfigKeys(
-      Configuration conf, String... keys) {
+  public static Optional<String> getHostNameFromConfigKeys(Configuration conf,
+      String... keys) {
     for (final String key : keys) {
       final String value = conf.getTrimmed(key);
       final Optional<String> hostName = getHostName(value);
@@ -621,13 +621,29 @@ public final class OzoneClientUtils {
    * @param addr configured address
    * @param rpcServer started RPC server.
    */
-  public static InetSocketAddress updateListenAddress(
+  public static InetSocketAddress updateRPCListenAddress(
       OzoneConfiguration conf, String rpcAddressKey,
       InetSocketAddress addr, RPC.Server rpcServer) {
-    InetSocketAddress listenAddr = rpcServer.getListenerAddress();
-    InetSocketAddress updatedAddr = new InetSocketAddress(
-        addr.getHostString(), listenAddr.getPort());
-    conf.set(rpcAddressKey,
+    return updateListenAddress(conf, rpcAddressKey, addr,
+        rpcServer.getListenerAddress());
+  }
+
+  /**
+   * After starting an server, updates configuration with the actual
+   * listening address of that server. The listening address may be different
+   * from the configured address if, for example, the configured address uses
+   * port 0 to request use of an ephemeral port.
+   *
+   * @param conf       configuration to update
+   * @param addressKey configuration key for RPC server address
+   * @param addr       configured address
+   * @param listenAddr the real listening address.
+   */
+  public static InetSocketAddress updateListenAddress(OzoneConfiguration conf,
+      String addressKey, InetSocketAddress addr, InetSocketAddress listenAddr) {
+    InetSocketAddress updatedAddr = new InetSocketAddress(addr.getHostString(),
+        listenAddr.getPort());
+    conf.set(addressKey,
         listenAddr.getHostString() + ":" + listenAddr.getPort());
     return updatedAddr;
   }

+ 3 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/OzoneConfigKeys.java

@@ -95,6 +95,9 @@ public final class OzoneConfigKeys {
   public static final String DFS_CONTAINER_RATIS_DATANODE_STORAGE_DIR =
       "dfs.container.ratis.datanode.storage.dir";
 
+  public static final String OZONE_SCM_WEB_AUTHENTICATION_KERBEROS_PRINCIPAL =
+      "ozone.web.authentication.kerberos.principal";
+  
   /**
    * There is no need to instantiate this class.
    */

+ 16 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/ksm/KSMConfigKeys.java

@@ -39,6 +39,22 @@ public final class KSMConfigKeys {
       "0.0.0.0";
   public static final int OZONE_KSM_PORT_DEFAULT = 9862;
 
+  public static final String OZONE_KSM_HTTP_ENABLED_KEY =
+      "ozone.ksm.http.enabled";
+  public static final String OZONE_KSM_HTTP_BIND_HOST_KEY =
+      "ozone.ksm.http-bind-host";
+  public static final String OZONE_KSM_HTTPS_BIND_HOST_KEY =
+      "ozone.ksm.https-bind-host";
+  public static final String OZONE_KSM_HTTP_ADDRESS_KEY =
+      "ozone.ksm.http-address";
+  public static final String OZONE_KSM_HTTPS_ADDRESS_KEY =
+      "ozone.ksm.https-address";
+  public static final String OZONE_KSM_KEYTAB_FILE =
+      "ozone.ksm.keytab.file";
+  public static final String OZONE_KSM_HTTP_BIND_HOST_DEFAULT = "0.0.0.0";
+  public static final int OZONE_KSM_HTTP_BIND_PORT_DEFAULT = 9874;
+  public static final int OZONE_KSM_HTTPS_BIND_PORT_DEFAULT = 9875;
+
   // LevelDB cache file uses an off-heap cache in LevelDB of 128 MB.
   public static final String OZONE_KSM_DB_CACHE_SIZE_MB =
       "ozone.ksm.leveldb.cache.size.mb";

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

@@ -76,6 +76,7 @@ public class KeySpaceManager implements KeySpaceManagerProtocol {
   private final BucketManager bucketManager;
   private final KeyManager keyManager;
   private final KSMMetrics metrics;
+  private final KeySpaceManagerHttpServer httpServer;
 
   public KeySpaceManager(OzoneConfiguration conf) throws IOException {
     final int handlerCount = conf.getInt(OZONE_KSM_HANDLER_COUNT_KEY,
@@ -91,13 +92,14 @@ public class KeySpaceManager implements KeySpaceManagerProtocol {
     ksmRpcServer = startRpcServer(conf, ksmNodeRpcAddr,
         KeySpaceManagerProtocolPB.class, ksmService,
         handlerCount);
-    ksmRpcAddress = OzoneClientUtils.updateListenAddress(conf,
+    ksmRpcAddress = OzoneClientUtils.updateRPCListenAddress(conf,
         OZONE_KSM_ADDRESS_KEY, ksmNodeRpcAddr, ksmRpcServer);
     metadataManager = new MetadataManagerImpl(conf);
     volumeManager = new VolumeManagerImpl(metadataManager, conf);
     bucketManager = new BucketManagerImpl(metadataManager);
     metrics = KSMMetrics.create();
     keyManager = new KeyManagerImpl(getScmBlockClient(conf), metadataManager);
+    httpServer = new KeySpaceManagerHttpServer(conf);
   }
 
   /**
@@ -193,11 +195,12 @@ public class KeySpaceManager implements KeySpaceManagerProtocol {
   /**
    * Start service.
    */
-  public void start() {
+  public void start() throws IOException {
     LOG.info(buildRpcServerStartMessage("KeyspaceManager RPC server",
         ksmRpcAddress));
     metadataManager.start();
     ksmRpcServer.start();
+    httpServer.start();
   }
 
   /**
@@ -207,8 +210,9 @@ public class KeySpaceManager implements KeySpaceManagerProtocol {
     try {
       ksmRpcServer.stop();
       metadataManager.stop();
-    } catch (IOException e) {
-      LOG.info("Key Space Manager stop failed.", e);
+      httpServer.stop();
+    } catch (Exception e) {
+      LOG.error("Key Space Manager stop failed.", e);
     }
   }
 

+ 74 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/ksm/KeySpaceManagerHttpServer.java

@@ -0,0 +1,74 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.ozone.ksm;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.web.OzoneHttpServer;
+
+import java.io.IOException;
+
+/**
+ * HttpServer wrapper for the KeySpaceManager.
+ */
+public class KeySpaceManagerHttpServer extends OzoneHttpServer {
+
+  public KeySpaceManagerHttpServer(Configuration conf) throws IOException {
+    super(conf, "ksm");
+  }
+
+  @Override protected String getHttpAddressKey() {
+    return KSMConfigKeys.OZONE_KSM_HTTP_ADDRESS_KEY;
+  }
+
+  @Override protected String getHttpBindHostKey() {
+    return KSMConfigKeys.OZONE_KSM_HTTP_BIND_HOST_KEY;
+  }
+
+  @Override protected String getHttpsAddressKey() {
+    return KSMConfigKeys.OZONE_KSM_HTTPS_ADDRESS_KEY;
+  }
+
+  @Override protected String getHttpsBindHostKey() {
+    return KSMConfigKeys.OZONE_KSM_HTTPS_BIND_HOST_KEY;
+  }
+
+  @Override protected String getBindHostDefault() {
+    return KSMConfigKeys.OZONE_KSM_HTTP_BIND_HOST_DEFAULT;
+  }
+
+  @Override protected int getHttpBindPortDefault() {
+    return KSMConfigKeys.OZONE_KSM_HTTP_BIND_PORT_DEFAULT;
+  }
+
+  @Override protected int getHttpsBindPortDefault() {
+    return KSMConfigKeys.OZONE_KSM_HTTPS_BIND_PORT_DEFAULT;
+  }
+
+  @Override protected String getKeytabFile() {
+    return KSMConfigKeys.OZONE_KSM_KEYTAB_FILE;
+  }
+
+  @Override protected String getSpnegoPrincipal() {
+    return OzoneConfigKeys.OZONE_SCM_WEB_AUTHENTICATION_KERBEROS_PRINCIPAL;
+  }
+
+  @Override protected String getEnabledKey() {
+    return KSMConfigKeys.OZONE_KSM_HTTP_ENABLED_KEY;
+  }
+}

+ 33 - 11
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/StorageContainerManager.java

@@ -158,6 +158,8 @@ public class StorageContainerManager
   private final RPC.Server blockRpcServer;
   private final InetSocketAddress blockRpcAddress;
 
+  private final StorageContainerManagerHttpServer httpServer;
+
   /** SCM mxbean. */
   private ObjectName scmInfoBeanName;
 
@@ -208,7 +210,7 @@ public class StorageContainerManager
     datanodeRpcServer = startRpcServer(conf, datanodeRpcAddr,
         StorageContainerDatanodeProtocolPB.class, dnProtoPbService,
         handlerCount);
-    datanodeRpcAddress = OzoneClientUtils.updateListenAddress(conf,
+    datanodeRpcAddress = OzoneClientUtils.updateRPCListenAddress(conf,
         OZONE_SCM_DATANODE_ADDRESS_KEY, datanodeRpcAddr, datanodeRpcServer);
 
     // SCM Container Service RPC
@@ -223,10 +225,9 @@ public class StorageContainerManager
     clientRpcServer = startRpcServer(conf, scmAddress,
         StorageContainerLocationProtocolPB.class, storageProtoPbService,
         handlerCount);
-    clientRpcAddress = OzoneClientUtils.updateListenAddress(conf,
+    clientRpcAddress = OzoneClientUtils.updateRPCListenAddress(conf,
         OZONE_SCM_CLIENT_ADDRESS_KEY, scmAddress, clientRpcServer);
 
-
     // SCM Block Service RPC
     BlockingService blockProtoPbService =
         ScmBlockLocationProtocolProtos
@@ -239,9 +240,11 @@ public class StorageContainerManager
     blockRpcServer = startRpcServer(conf, scmBlockAddress,
         ScmBlockLocationProtocolPB.class, blockProtoPbService,
         handlerCount);
-    blockRpcAddress = OzoneClientUtils.updateListenAddress(conf,
+    blockRpcAddress = OzoneClientUtils.updateRPCListenAddress(conf,
         OZONE_SCM_BLOCK_CLIENT_ADDRESS_KEY, scmBlockAddress, blockRpcServer);
 
+    httpServer = new StorageContainerManagerHttpServer(conf);
+
     registerMXBean();
   }
 
@@ -459,7 +462,7 @@ public class StorageContainerManager
   /**
    * Start service.
    */
-  public void start() {
+  public void start() throws IOException {
     LOG.info(buildRpcServerStartMessage(
         "StorageContainerLocationProtocol RPC server", clientRpcAddress));
     clientRpcServer.start();
@@ -469,18 +472,36 @@ public class StorageContainerManager
     LOG.info(buildRpcServerStartMessage("RPC server for DataNodes",
         datanodeRpcAddress));
     datanodeRpcServer.start();
+
+    httpServer.start();
+
+
   }
 
   /**
    * Stop service.
    */
   public void stop() {
-    LOG.info("Stopping block service RPC server");
-    blockRpcServer.stop();
-    LOG.info("Stopping the StorageContainerLocationProtocol RPC server");
-    clientRpcServer.stop();
-    LOG.info("Stopping the RPC server for DataNodes");
-    datanodeRpcServer.stop();
+    try {
+      LOG.info("Stopping block service RPC server");
+      blockRpcServer.stop();
+    } catch (Exception ex) {
+      LOG.error("Storage Container Manager blockRpcServer stop failed.", ex);
+    }
+
+    try {
+      LOG.info("Stopping the StorageContainerLocationProtocol RPC server");
+      clientRpcServer.stop();
+    } catch (Exception ex) {
+      LOG.error("Storage Container Manager clientRpcServer stop failed.", ex);
+    }
+
+    try {
+      LOG.info("Stopping the RPC server for DataNodes");
+      datanodeRpcServer.stop();
+    } catch (Exception ex) {
+      LOG.error("Storage Container Manager httpServer stop failed.", ex);
+    }
     unregisterMXBean();
     IOUtils.closeQuietly(scmContainerManager);
     IOUtils.closeQuietly(scmBlockManager);
@@ -676,4 +697,5 @@ public class StorageContainerManager
       }
     }
   }
+
 }

+ 77 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/scm/StorageContainerManagerHttpServer.java

@@ -0,0 +1,77 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.ozone.scm;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.ozone.OzoneConfigKeys;
+import org.apache.hadoop.ozone.web.OzoneHttpServer;
+import org.apache.hadoop.scm.ScmConfigKeys;
+
+import java.io.IOException;
+
+/**
+ * HttpServer2 wrapper for the Ozone Storage Container Manager.
+ */
+public class StorageContainerManagerHttpServer extends OzoneHttpServer {
+
+  public StorageContainerManagerHttpServer(Configuration conf)
+      throws IOException {
+    super(conf, "scm");
+  }
+
+  @Override protected String getHttpAddressKey() {
+    return ScmConfigKeys.OZONE_SCM_HTTP_ADDRESS_KEY;
+  }
+
+  @Override protected String getHttpBindHostKey() {
+    return ScmConfigKeys.OZONE_SCM_HTTP_BIND_HOST_KEY;
+  }
+
+  @Override protected String getHttpsAddressKey() {
+    return ScmConfigKeys.OZONE_SCM_HTTPS_ADDRESS_KEY;
+  }
+
+  @Override protected String getHttpsBindHostKey() {
+    return ScmConfigKeys.OZONE_SCM_HTTPS_BIND_HOST_KEY;
+  }
+
+  @Override protected String getBindHostDefault() {
+    return ScmConfigKeys.OZONE_SCM_HTTP_BIND_HOST_DEFAULT;
+  }
+
+  @Override protected int getHttpBindPortDefault() {
+    return ScmConfigKeys.OZONE_SCM_HTTP_BIND_PORT_DEFAULT;
+  }
+
+  @Override protected int getHttpsBindPortDefault() {
+    return ScmConfigKeys.OZONE_SCM_HTTPS_BIND_PORT_DEFAULT;
+  }
+
+  @Override protected String getKeytabFile() {
+    return ScmConfigKeys.OZONE_SCM_KEYTAB_FILE;
+  }
+
+  @Override protected String getSpnegoPrincipal() {
+    return OzoneConfigKeys.OZONE_SCM_WEB_AUTHENTICATION_KERBEROS_PRINCIPAL;
+  }
+
+  @Override protected String getEnabledKey() {
+    return ScmConfigKeys.OZONE_SCM_HTTP_ENABLED_KEY;
+  }
+
+}

+ 178 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/web/OzoneHttpServer.java

@@ -0,0 +1,178 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with this
+ * work for additional information regarding copyright ownership.  The ASF
+ * licenses this file to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance with the License.
+ * You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS, WITHOUT
+ * WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. See the
+ * License for the specific language governing permissions and limitations under
+ * the License.
+ */
+
+package org.apache.hadoop.ozone.web;
+
+import com.google.common.base.Optional;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hdfs.DFSUtil;
+import org.apache.hadoop.http.HttpConfig;
+import org.apache.hadoop.http.HttpServer2;
+import org.apache.hadoop.net.NetUtils;
+import org.apache.hadoop.ozone.OzoneClientUtils;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.net.InetSocketAddress;
+
+/**
+ * Base class for HTTP server of the Ozone related components.
+ */
+public abstract class OzoneHttpServer {
+
+  private static final Logger LOG =
+      LoggerFactory.getLogger(OzoneHttpServer.class);
+
+  private HttpServer2 httpServer;
+  private final Configuration conf;
+
+  private InetSocketAddress httpAddress;
+  private InetSocketAddress httpsAddress;
+
+  private HttpConfig.Policy policy;
+
+  private String name;
+
+  public OzoneHttpServer(Configuration conf, String name) throws IOException {
+    this.name = name;
+    this.conf = conf;
+    if (isEnabled()) {
+      policy = DFSUtil.getHttpPolicy(conf);
+      if (policy.isHttpEnabled()) {
+        this.httpAddress = getHttpBindAddress();
+      }
+      if (policy.isHttpsEnabled()) {
+        this.httpsAddress = getHttpsBindAddress();
+      }
+      HttpServer2.Builder builder = null;
+      builder = DFSUtil.httpServerTemplateForNNAndJN(conf, this.httpAddress,
+          this.httpsAddress, name, getSpnegoPrincipal(), getKeytabFile());
+      httpServer = builder.build();
+
+    }
+
+  }
+
+  protected InetSocketAddress getBindAddress(String bindHostKey,
+      String addressKey, String bindHostDefault, int bindPortdefault) {
+    final Optional<String> bindHost =
+        OzoneClientUtils.getHostNameFromConfigKeys(conf, bindHostKey);
+
+    final Optional<Integer> addressPort =
+        OzoneClientUtils.getPortNumberFromConfigKeys(conf, addressKey);
+
+    final Optional<String> addresHost =
+        OzoneClientUtils.getHostNameFromConfigKeys(conf, addressKey);
+
+    String hostName = bindHost.or(addresHost).or(bindHostDefault);
+
+    return NetUtils.createSocketAddr(
+        hostName + ":" + addressPort.or(bindPortdefault));
+  }
+
+  /**
+   * Retrieve the socket address that should be used by clients to connect
+   * to the  HTTPS web interface.
+   *
+   * @return Target InetSocketAddress for the Ozone HTTPS endpoint.
+   */
+  public InetSocketAddress getHttpsBindAddress() {
+    return getBindAddress(getHttpsBindHostKey(), getHttpsAddressKey(),
+        getBindHostDefault(), getHttpsBindPortDefault());
+  }
+
+  /**
+   * Retrieve the socket address that should be used by clients to connect
+   * to the  HTTP web interface.
+   * <p>
+   * * @return Target InetSocketAddress for the Ozone HTTP endpoint.
+   */
+  public InetSocketAddress getHttpBindAddress() {
+    return getBindAddress(getHttpBindHostKey(), getHttpAddressKey(),
+        getBindHostDefault(), getHttpBindPortDefault());
+
+  }
+
+  public void start() throws IOException {
+    if (httpServer != null && isEnabled()) {
+      httpServer.start();
+      updateConnectorAddress();
+    }
+
+  }
+
+  private boolean isEnabled() {
+    return conf.getBoolean(getEnabledKey(), true);
+  }
+
+  public void stop() throws Exception {
+    if (httpServer != null) {
+      httpServer.stop();
+    }
+  }
+
+  /**
+   * Update the configured listen address based on the real port
+   * <p>
+   * (eg. replace :0 with real port)
+   */
+  public void updateConnectorAddress() {
+    int connIdx = 0;
+    if (policy.isHttpEnabled()) {
+      httpAddress = httpServer.getConnectorAddress(connIdx++);
+      String realAddress = NetUtils.getHostPortString(httpAddress);
+      conf.set(getHttpAddressKey(), realAddress);
+      LOG.info(
+          String.format("HTTP server of SCM is listening at %s", realAddress));
+    }
+
+    if (policy.isHttpsEnabled()) {
+      httpsAddress = httpServer.getConnectorAddress(connIdx);
+      conf.set(getHttpsAddressKey(), NetUtils.getHostPortString(httpsAddress));
+    }
+  }
+
+  public InetSocketAddress getHttpAddress() {
+    return httpAddress;
+  }
+
+  public InetSocketAddress getHttpsAddress() {
+    return httpsAddress;
+  }
+
+  protected abstract String getHttpAddressKey();
+
+  protected abstract String getHttpsAddressKey();
+
+  protected abstract String getHttpBindHostKey();
+
+  protected abstract String getHttpsBindHostKey();
+
+  protected abstract String getBindHostDefault();
+
+  protected abstract int getHttpBindPortDefault();
+
+  protected abstract int getHttpsBindPortDefault();
+
+  protected abstract String getKeytabFile();
+
+  protected abstract String getSpnegoPrincipal();
+
+  protected abstract String getEnabledKey();
+
+}

+ 24 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/ozone/web/package-info.java

@@ -0,0 +1,24 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.ozone.web;
+
+/**
+ * This package contains generic class for the internal http server
+ * and REST interfaces.
+ */

+ 129 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/resources/ozone-default.xml

@@ -451,4 +451,133 @@
     </description>
   </property>
 
+
+  <property>
+    <name>ozone.scm.http.enabled</name>
+    <value>true</value>
+    <description>
+      Property to enable or disable SCM web ui.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.scm.http-address</name>
+    <value>0.0.0.0:9876</value>
+    <description>
+      The address and the base port where the SCM web ui will listen on.
+
+      If the port is 0 then the server will start on a free port.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.scm.http-bind-host</name>
+    <value>0.0.0.0</value>
+    <description>
+      The actual address the SCM web server will bind to. If this optional
+      address is set, it overrides only the hostname portion of
+      ozone.scm.http-address.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.scm.https-address</name>
+    <value>0.0.0.0:9877</value>
+    <description>
+      The address and the base port where the SCM web ui will listen on
+      using HTTPS.
+
+      If the port is 0 then the server will start on a free port.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.scm.https-bind-host</name>
+    <value>0.0.0.0</value>
+    <description>
+      The actual address the SCM web server will bind to using HTTPS. If this
+      optional address is set, it overrides only the hostname portion of
+      ozone.scm.http-address.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.ksm.http.enabled</name>
+    <value>true</value>
+    <description>
+      Property to enable or disable KSM web ui.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.ksm.http-address</name>
+    <value>0.0.0.0:9874</value>
+    <description>
+      The address and the base port where the KSM web ui will listen on.
+
+      If the port is 0 then the server will start on a free port.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.ksm.http-bind-host</name>
+    <value>0.0.0.0</value>
+    <description>
+      The actual address the KSM web server will bind to. If this optional
+      address is set, it overrides only the hostname portion of
+      ozone.ksm.http-address.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.ksm.https-address</name>
+    <value>0.0.0.0:9875</value>
+    <description>
+      The address and the base port where the KSM web ui will listen on
+      using HTTPS.
+
+      If the port is 0 then the server will start on a free port.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.ksm.https-bind-host</name>
+    <value>0.0.0.0</value>
+    <description>
+      The actual address the KSM web server will bind to using HTTPS. If this
+      optional address is set, it overrides only the hostname portion of
+      ozone.ksm.http-address.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.web.authentication.kerberos.principal</name>
+    <value></value>
+    <description>
+      The server principal used by the SCM and KSM for web UI SPNEGO
+      authentication when Kerberos security is enabled. This is
+      typically set to HTTP/_HOST@REALM.TLD The SPNEGO server principal
+      begins with the prefix HTTP/ by convention.
+
+      If the value is '*', the web server will attempt to login with
+      every principal specified in the keytab file.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.scm.keytab.file</name>
+    <value></value>
+    <description>
+      The keytab file for Kerberos authentication in SCM.
+    </description>
+  </property>
+
+  <property>
+    <name>ozone.ksm.keytab.file</name>
+    <value></value>
+    <description>
+      The keytab file for Kerberos authentication in KSM.
+    </description>
+  </property>
+  
 </configuration>

+ 74 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/ksm/index.html

@@ -0,0 +1,74 @@
+<!DOCTYPE html PUBLIC "-//W3C//DTD XHTML 1.0 Strict//EN"
+        "http://www.w3.org/TR/xhtml1/DTD/xhtml1-strict.dtd">
+<!--
+   Licensed to the Apache Software Foundation (ASF) under one or more
+   contributor license agreements.  See the NOTICE file distributed with
+   this work for additional information regarding copyright ownership.
+   The ASF licenses this file to You under the Apache License, Version 2.0
+   (the "License"); you may not use this file except in compliance with
+   the License.  You may obtain a copy of the License at
+
+       http://www.apache.org/licenses/LICENSE-2.0
+
+   Unless required by applicable law or agreed to in writing, software
+   distributed under the License is distributed on an "AS IS" BASIS,
+   WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+   See the License for the specific language governing permissions and
+   limitations under the License.
+-->
+<html lang="en">
+<head>
+    <meta charset="utf-8">
+    <meta http-equiv="X-UA-Compatible" content="IE=edge">
+    <meta name="viewport" content="width=device-width, initial-scale=1">
+    <!-- The above 3 meta tags *must* come first in the head; any other head content must come *after* these tags -->
+    <meta name="description" content="HDFS Storage Container Manager">
+
+    <title>HDFS Storage Container Manager</title>
+
+    <link href="/static/bootstrap-3.0.2/css/bootstrap.min.css" rel="stylesheet">
+    <link href="/static/hadoop.css" rel="stylesheet">
+
+    <link href="/main.css" rel="stylesheet">
+
+</head>
+
+<body>
+
+<nav class="navbar navbar-inverse navbar-fixed-top">
+    <div class="container">
+        <div class="navbar-header">
+            <button type="button" class="navbar-toggle collapsed" data-toggle="collapse" data-target="#navbar"
+                    aria-expanded="false" aria-controls="navbar">
+                <span class="sr-only">Toggle navigation</span>
+                <span class="icon-bar"></span>
+                <span class="icon-bar"></span>
+                <span class="icon-bar"></span>
+            </button>
+            <a class="navbar-brand" href="#">HDFS KSM</a>
+        </div>
+        <div id="navbar" class="collapse navbar-collapse">
+            <ul class="nav navbar-nav">
+                <li><a href="#">Home</a></li>
+                <li><a href="/jmx">JMX</a></li>
+                <li><a href="/conf">Config</a></li>
+                <li><a href="/stacks">Stacks</a></li>
+                <li><a href="/logLevel">Log levels</a></li>
+
+            </ul>
+        </div><!--/.nav-collapse -->
+    </div>
+</nav>
+
+<div class="container">
+
+    <div class="starter-template">
+        <h1>HDFS KSM</h1>
+    </div>
+
+</div><!-- /.container -->
+
+<script src="/static/jquery-1.10.2.min.js"></script>
+<script src="/static/bootstrap-3.0.2//js/bootstrap.min.js"></script>
+</body>
+</html>

+ 23 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/ksm/main.css

@@ -0,0 +1,23 @@
+/**
+ *   Licensed to the Apache Software Foundation (ASF) under one or more
+ *  contributor license agreements.  See the NOTICE file distributed with
+ *  this work for additional information regarding copyright ownership.
+ *  The ASF licenses this file to You under the Apache License, Version 2.0
+ *  (the "License"); you may not use this file except in compliance with
+ *  the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+*/
+body {
+  padding-top: 50px;
+}
+.starter-template {
+  padding: 40px 15px;
+  text-align: center;
+}

+ 75 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/scm/index.html

@@ -0,0 +1,75 @@
+<!DOCTYPE html PUBLIC "-//W3C//DTD XHTML 1.0 Strict//EN"
+        "http://www.w3.org/TR/xhtml1/DTD/xhtml1-strict.dtd">
+<!--
+   Licensed to the Apache Software Foundation (ASF) under one or more
+   contributor license agreements.  See the NOTICE file distributed with
+   this work for additional information regarding copyright ownership.
+   The ASF licenses this file to You under the Apache License, Version 2.0
+   (the "License"); you may not use this file except in compliance with
+   the License.  You may obtain a copy of the License at
+
+       http://www.apache.org/licenses/LICENSE-2.0
+
+   Unless required by applicable law or agreed to in writing, software
+   distributed under the License is distributed on an "AS IS" BASIS,
+   WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+   See the License for the specific language governing permissions and
+   limitations under the License.
+-->
+<html lang="en">
+<head>
+    <meta charset="utf-8">
+    <meta http-equiv="X-UA-Compatible" content="IE=edge">
+    <meta name="viewport" content="width=device-width, initial-scale=1">
+    <!-- The above 3 meta tags *must* come first in the head; any other head content must come *after* these tags -->
+    <meta name="description" content="HDFS Storage Container Manager">
+
+    <title>HDFS Storage Container Manager</title>
+
+    <link href="/static/bootstrap-3.0.2/css/bootstrap.min.css" rel="stylesheet">
+    <link href="/static/hadoop.css" rel="stylesheet">
+
+    <link href="/main.css" rel="stylesheet">
+
+</head>
+
+<body>
+
+<nav class="navbar navbar-inverse navbar-fixed-top">
+    <div class="container">
+        <div class="navbar-header">
+            <button type="button" class="navbar-toggle collapsed" data-toggle="collapse" data-target="#navbar"
+                    aria-expanded="false" aria-controls="navbar">
+                <span class="sr-only">Toggle navigation</span>
+                <span class="icon-bar"></span>
+                <span class="icon-bar"></span>
+                <span class="icon-bar"></span>
+            </button>
+            <a class="navbar-brand" href="#">HDFS SCM</a>
+        </div>
+        <div id="navbar" class="collapse navbar-collapse">
+            <ul class="nav navbar-nav">
+                <li><a href="#">Home</a></li>
+                <li><a href="/jmx">JMX</a></li>
+                <li><a href="/conf">Config</a></li>
+                <li><a href="/stacks">Stacks</a></li>
+                <li><a href="/logLevel">Log levels</a></li>
+
+            </ul>
+        </div><!--/.nav-collapse -->
+    </div>
+</nav>
+
+<div class="container">
+
+    <div class="starter-template">
+        <h1>HDFS SCM</h1>
+    </div>
+
+</div><!-- /.container -->
+
+
+<script src="/static/jquery-1.10.2.min.js"></script>
+<script src="/static/bootstrap-3.0.2//js/bootstrap.min.js"></script>
+</body>
+</html>

+ 23 - 0
hadoop-hdfs-project/hadoop-hdfs/src/main/webapps/scm/main.css

@@ -0,0 +1,23 @@
+/**
+ *   Licensed to the Apache Software Foundation (ASF) under one or more
+ *  contributor license agreements.  See the NOTICE file distributed with
+ *  this work for additional information regarding copyright ownership.
+ *  The ASF licenses this file to You under the Apache License, Version 2.0
+ *  (the "License"); you may not use this file except in compliance with
+ *  the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+*/
+body {
+  padding-top: 50px;
+}
+.starter-template {
+  padding: 40px 15px;
+  text-align: center;
+}

+ 6 - 3
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNameNodeHttpServer.java

@@ -52,8 +52,10 @@ public class TestNameNodeHttpServer {
 
   @Parameters
   public static Collection<Object[]> policy() {
-    Object[][] params = new Object[][] { { HttpConfig.Policy.HTTP_ONLY },
-        { HttpConfig.Policy.HTTPS_ONLY }, { HttpConfig.Policy.HTTP_AND_HTTPS } };
+    Object[][] params = new Object[][] {
+        {HttpConfig.Policy.HTTP_ONLY },
+        {HttpConfig.Policy.HTTPS_ONLY },
+        {HttpConfig.Policy.HTTP_AND_HTTPS } };
     return Arrays.asList(params);
   }
 
@@ -116,8 +118,9 @@ public class TestNameNodeHttpServer {
   }
 
   private static boolean canAccess(String scheme, InetSocketAddress addr) {
-    if (addr == null)
+    if (addr == null) {
       return false;
+    }
     try {
       URL url = new URL(scheme + "://" + NetUtils.getHostPortString(addr));
       URLConnection conn = connectionFactory.openConnection(url);

+ 4 - 1
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/MiniOzoneCluster.java

@@ -365,7 +365,9 @@ public final class MiniOzoneCluster extends MiniDFSCluster
       conf.set(ScmConfigKeys.OZONE_SCM_CLIENT_ADDRESS_KEY, "127.0.0.1:0");
       conf.set(ScmConfigKeys.OZONE_SCM_BLOCK_CLIENT_ADDRESS_KEY, "127.0.0.1:0");
       conf.set(ScmConfigKeys.OZONE_SCM_DATANODE_ADDRESS_KEY, "127.0.0.1:0");
+      conf.set(ScmConfigKeys.OZONE_SCM_HTTP_ADDRESS_KEY, "127.0.0.1:0");
       conf.set(KSMConfigKeys.OZONE_KSM_ADDRESS_KEY, "127.0.0.1:0");
+      conf.set(KSMConfigKeys.OZONE_KSM_HTTP_ADDRESS_KEY, "127.0.0.1:0");
 
       // Use random ports for ozone containers in mini cluster,
       // in order to launch multiple container servers per node.
@@ -431,7 +433,8 @@ public final class MiniOzoneCluster extends MiniDFSCluster
         throw new IllegalArgumentException(
             "The Ozone handler type must be specified.");
       } else {
-        conf.set(OzoneConfigKeys.OZONE_HANDLER_TYPE_KEY, ozoneHandlerType.get());
+        conf.set(OzoneConfigKeys.OZONE_HANDLER_TYPE_KEY,
+            ozoneHandlerType.get());
       }
     }
 

+ 141 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/ksm/TestKeySpaceManagerHttpServer.java

@@ -0,0 +1,141 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.ozone.ksm;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.FileUtil;
+import org.apache.hadoop.hdfs.DFSConfigKeys;
+import org.apache.hadoop.hdfs.web.URLConnectionFactory;
+import org.apache.hadoop.http.HttpConfig;
+import org.apache.hadoop.http.HttpConfig.Policy;
+import org.apache.hadoop.net.NetUtils;
+import org.apache.hadoop.scm.ScmConfigKeys;
+import org.apache.hadoop.security.ssl.KeyStoreTestUtil;
+import org.apache.hadoop.test.GenericTestUtils;
+import org.junit.AfterClass;
+import org.junit.Assert;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.junit.runners.Parameterized;
+import org.junit.runners.Parameterized.Parameters;
+
+import java.io.File;
+import java.net.InetSocketAddress;
+import java.net.URL;
+import java.net.URLConnection;
+import java.util.Arrays;
+import java.util.Collection;
+
+/**
+ * Test http server os KSM with various HTTP option.
+ */
+@RunWith(value = Parameterized.class)
+public class TestKeySpaceManagerHttpServer {
+  private static final String BASEDIR = GenericTestUtils
+      .getTempPath(TestKeySpaceManagerHttpServer.class.getSimpleName());
+  private static String keystoresDir;
+  private static String sslConfDir;
+  private static Configuration conf;
+  private static URLConnectionFactory connectionFactory;
+
+  @Parameters public static Collection<Object[]> policy() {
+    Object[][] params = new Object[][] {
+        {HttpConfig.Policy.HTTP_ONLY},
+        {HttpConfig.Policy.HTTPS_ONLY},
+        {HttpConfig.Policy.HTTP_AND_HTTPS} };
+    return Arrays.asList(params);
+  }
+
+  private final HttpConfig.Policy policy;
+
+  public TestKeySpaceManagerHttpServer(Policy policy) {
+    super();
+    this.policy = policy;
+  }
+
+  @BeforeClass public static void setUp() throws Exception {
+    File base = new File(BASEDIR);
+    FileUtil.fullyDelete(base);
+    base.mkdirs();
+    conf = new Configuration();
+    keystoresDir = new File(BASEDIR).getAbsolutePath();
+    sslConfDir = KeyStoreTestUtil.getClasspathDir(
+        org.apache.hadoop.hdfs.server.namenode.TestNameNodeHttpServer.class);
+    KeyStoreTestUtil.setupSSLConfig(keystoresDir, sslConfDir, conf, false);
+    connectionFactory =
+        URLConnectionFactory.newDefaultURLConnectionFactory(conf);
+    conf.set(DFSConfigKeys.DFS_CLIENT_HTTPS_KEYSTORE_RESOURCE_KEY,
+        KeyStoreTestUtil.getClientSSLConfigFileName());
+    conf.set(DFSConfigKeys.DFS_SERVER_HTTPS_KEYSTORE_RESOURCE_KEY,
+        KeyStoreTestUtil.getServerSSLConfigFileName());
+  }
+
+  @AfterClass public static void tearDown() throws Exception {
+    FileUtil.fullyDelete(new File(BASEDIR));
+    KeyStoreTestUtil.cleanupSSLConfig(keystoresDir, sslConfDir);
+  }
+
+  @Test public void testHttpPolicy() throws Exception {
+    conf.set(DFSConfigKeys.DFS_HTTP_POLICY_KEY, policy.name());
+    conf.set(ScmConfigKeys.OZONE_SCM_HTTPS_ADDRESS_KEY, "localhost:0");
+
+    InetSocketAddress addr = InetSocketAddress.createUnresolved("localhost", 0);
+    KeySpaceManagerHttpServer server = null;
+    try {
+      server = new KeySpaceManagerHttpServer(conf);
+      server.start();
+
+      Assert.assertTrue(implies(policy.isHttpEnabled(),
+          canAccess("http", server.getHttpAddress())));
+      Assert.assertTrue(
+          implies(!policy.isHttpEnabled(), server.getHttpAddress() == null));
+
+      Assert.assertTrue(implies(policy.isHttpsEnabled(),
+          canAccess("https", server.getHttpsAddress())));
+      Assert.assertTrue(
+          implies(!policy.isHttpsEnabled(), server.getHttpsAddress() == null));
+
+    } finally {
+      if (server != null) {
+        server.stop();
+      }
+    }
+  }
+
+  private static boolean canAccess(String scheme, InetSocketAddress addr) {
+    if (addr == null) {
+      return false;
+    }
+    try {
+      URL url =
+          new URL(scheme + "://" + NetUtils.getHostPortString(addr) + "/jmx");
+      URLConnection conn = connectionFactory.openConnection(url);
+      conn.connect();
+      conn.getContent();
+    } catch (Exception e) {
+      return false;
+    }
+    return true;
+  }
+
+  private static boolean implies(boolean a, boolean b) {
+    return !a || b;
+  }
+}

+ 21 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/ksm/package-info.java

@@ -0,0 +1,21 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ *  with the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ */
+package org.apache.hadoop.ozone.ksm;
+/**
+ * KSM tests
+ */

+ 141 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/scm/TestStorageContainerManagerHttpServer.java

@@ -0,0 +1,141 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ * <p>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.ozone.scm;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.FileUtil;
+import org.apache.hadoop.hdfs.DFSConfigKeys;
+import org.apache.hadoop.hdfs.web.URLConnectionFactory;
+import org.apache.hadoop.http.HttpConfig;
+import org.apache.hadoop.http.HttpConfig.Policy;
+import org.apache.hadoop.net.NetUtils;
+import org.apache.hadoop.scm.ScmConfigKeys;
+import org.apache.hadoop.security.ssl.KeyStoreTestUtil;
+import org.apache.hadoop.test.GenericTestUtils;
+import org.junit.AfterClass;
+import org.junit.Assert;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.junit.runner.RunWith;
+import org.junit.runners.Parameterized;
+import org.junit.runners.Parameterized.Parameters;
+
+import java.io.File;
+import java.net.InetSocketAddress;
+import java.net.URL;
+import java.net.URLConnection;
+import java.util.Arrays;
+import java.util.Collection;
+
+/**
+ * Test http server os SCM with various HTTP option.
+ */
+@RunWith(value = Parameterized.class)
+public class TestStorageContainerManagerHttpServer {
+  private static final String BASEDIR = GenericTestUtils
+      .getTempPath(TestStorageContainerManagerHttpServer.class.getSimpleName());
+  private static String keystoresDir;
+  private static String sslConfDir;
+  private static Configuration conf;
+  private static URLConnectionFactory connectionFactory;
+
+  @Parameters public static Collection<Object[]> policy() {
+    Object[][] params = new Object[][] {
+        {HttpConfig.Policy.HTTP_ONLY},
+        {HttpConfig.Policy.HTTPS_ONLY},
+        {HttpConfig.Policy.HTTP_AND_HTTPS} };
+    return Arrays.asList(params);
+  }
+
+  private final HttpConfig.Policy policy;
+
+  public TestStorageContainerManagerHttpServer(Policy policy) {
+    super();
+    this.policy = policy;
+  }
+
+  @BeforeClass public static void setUp() throws Exception {
+    File base = new File(BASEDIR);
+    FileUtil.fullyDelete(base);
+    base.mkdirs();
+    conf = new Configuration();
+    keystoresDir = new File(BASEDIR).getAbsolutePath();
+    sslConfDir = KeyStoreTestUtil.getClasspathDir(
+        org.apache.hadoop.hdfs.server.namenode.TestNameNodeHttpServer.class);
+    KeyStoreTestUtil.setupSSLConfig(keystoresDir, sslConfDir, conf, false);
+    connectionFactory =
+        URLConnectionFactory.newDefaultURLConnectionFactory(conf);
+    conf.set(DFSConfigKeys.DFS_CLIENT_HTTPS_KEYSTORE_RESOURCE_KEY,
+        KeyStoreTestUtil.getClientSSLConfigFileName());
+    conf.set(DFSConfigKeys.DFS_SERVER_HTTPS_KEYSTORE_RESOURCE_KEY,
+        KeyStoreTestUtil.getServerSSLConfigFileName());
+  }
+
+  @AfterClass public static void tearDown() throws Exception {
+    FileUtil.fullyDelete(new File(BASEDIR));
+    KeyStoreTestUtil.cleanupSSLConfig(keystoresDir, sslConfDir);
+  }
+
+  @Test public void testHttpPolicy() throws Exception {
+    conf.set(DFSConfigKeys.DFS_HTTP_POLICY_KEY, policy.name());
+    conf.set(ScmConfigKeys.OZONE_SCM_HTTPS_ADDRESS_KEY, "localhost:0");
+
+    InetSocketAddress addr = InetSocketAddress.createUnresolved("localhost", 0);
+    StorageContainerManagerHttpServer server = null;
+    try {
+      server = new StorageContainerManagerHttpServer(conf);
+      server.start();
+
+      Assert.assertTrue(implies(policy.isHttpEnabled(),
+          canAccess("http", server.getHttpAddress())));
+      Assert.assertTrue(
+          implies(!policy.isHttpEnabled(), server.getHttpAddress() == null));
+
+      Assert.assertTrue(implies(policy.isHttpsEnabled(),
+          canAccess("https", server.getHttpsAddress())));
+      Assert.assertTrue(
+          implies(!policy.isHttpsEnabled(), server.getHttpsAddress() == null));
+
+    } finally {
+      if (server != null) {
+        server.stop();
+      }
+    }
+  }
+
+  private static boolean canAccess(String scheme, InetSocketAddress addr) {
+    if (addr == null) {
+      return false;
+    }
+    try {
+      URL url =
+          new URL(scheme + "://" + NetUtils.getHostPortString(addr) + "/jmx");
+      URLConnection conn = connectionFactory.openConnection(url);
+      conn.connect();
+      conn.getContent();
+    } catch (Exception e) {
+      return false;
+    }
+    return true;
+  }
+
+  private static boolean implies(boolean a, boolean b) {
+    return !a || b;
+  }
+}

+ 98 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/web/TestOzoneHttpServer.java

@@ -0,0 +1,98 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.ozone.web;
+
+import org.apache.hadoop.ozone.OzoneConfiguration;
+import org.junit.Assert;
+import org.junit.Test;
+
+/**
+ * Test Common ozone web methods.
+ */
+public class TestOzoneHttpServer {
+  @Test
+  public void getBindAddress() throws Exception {
+    OzoneConfiguration conf = new OzoneConfiguration();
+    conf.set("enabled", "false");
+
+    OzoneHttpServer ozoneHttpServer = new OzoneHttpServer(conf, "test") {
+      @Override
+      protected String getHttpAddressKey() {
+        return null;
+      }
+
+      @Override
+      protected String getHttpsAddressKey() {
+        return null;
+      }
+
+      @Override
+      protected String getHttpBindHostKey() {
+        return null;
+      }
+
+      @Override
+      protected String getHttpsBindHostKey() {
+        return null;
+      }
+
+      @Override
+      protected String getBindHostDefault() {
+        return null;
+      }
+
+      @Override
+      protected int getHttpBindPortDefault() {
+        return 0;
+      }
+
+      @Override
+      protected int getHttpsBindPortDefault() {
+        return 0;
+      }
+
+      @Override
+      protected String getKeytabFile() {
+        return null;
+      }
+
+      @Override
+      protected String getSpnegoPrincipal() {
+        return null;
+      }
+
+      @Override
+      protected String getEnabledKey() {
+        return "enabled";
+      }
+    };
+
+    conf.set("addresskey", "0.0.0.0:1234");
+
+    Assert.assertEquals("/0.0.0.0:1234", ozoneHttpServer
+        .getBindAddress("bindhostkey", "addresskey",
+            "default", 65).toString());
+
+    conf.set("bindhostkey", "1.2.3.4");
+
+    Assert.assertEquals("/1.2.3.4:1234", ozoneHttpServer
+        .getBindAddress("bindhostkey", "addresskey",
+            "default", 65).toString());
+  }
+
+}

+ 21 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/ozone/web/package-info.java

@@ -0,0 +1,21 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ *  with the License.  You may obtain a copy of the License at
+ *
+ *      http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ */
+package org.apache.hadoop.ozone.web;
+/**
+ * Unit tests of generic ozone web app and rest utils.
+ */