Browse Source

HADOOP-3824. Move base functionality of StatusHttpServer to a core package.
Contributed by TszWo (Nicholas), SZE


git-svn-id: https://svn.apache.org/repos/asf/hadoop/core/trunk@680577 13f79535-47bb-0310-9956-ffa450edef68

Christopher Douglas 17 years ago
parent
commit
e6eea4fdca

+ 3 - 0
CHANGES.txt

@@ -115,6 +115,9 @@ Trunk (unreleased changes)
     HADOOP-3169. LeaseChecker daemon should not be started in DFSClient
     constructor. (TszWo (Nicholas), SZE via hairong)
 
+    HADOOP-3824. Move base functionality of StatusHttpServer to a core
+    package. (TszWo (Nicholas), SZE via cdouglas)
+
   OPTIMIZATIONS
 
     HADOOP-3556. Removed lock contention in MD5Hash by changing the 

+ 256 - 0
src/core/org/apache/hadoop/http/HttpServer.java

@@ -0,0 +1,256 @@
+/**
+ * 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.http;
+
+import java.io.IOException;
+import java.io.PrintWriter;
+import java.net.URL;
+import java.net.InetSocketAddress;
+import javax.servlet.ServletException;
+import javax.servlet.http.HttpServlet;
+import javax.servlet.http.HttpServletRequest;
+import javax.servlet.http.HttpServletResponse;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.log.LogLevel;
+import org.apache.hadoop.util.ReflectionUtils;
+
+import org.mortbay.http.HttpContext;
+import org.mortbay.http.SocketListener;
+import org.mortbay.http.SslListener;
+import org.mortbay.http.handler.ResourceHandler;
+import org.mortbay.jetty.servlet.WebApplicationContext;
+
+/**
+ * Create a Jetty embedded server to answer http requests. The primary goal
+ * is to serve up status information for the server.
+ * There are three contexts:
+ *   "/logs/" -> points to the log directory
+ *   "/static/" -> points to common static files (src/webapps/static)
+ *   "/" -> the jsp server code from (src/webapps/<name>)
+ */
+public class HttpServer {
+  public static final Log LOG = LogFactory.getLog(HttpServer.class);
+
+  protected final org.mortbay.jetty.Server webServer;
+  protected final WebApplicationContext webAppContext;
+  protected final boolean findPort;
+  protected final SocketListener listener;
+  private SslListener sslListener;
+
+  /**
+   * Create a status server on the given port.
+   * The jsp scripts are taken from src/webapps/<name>.
+   * @param name The name of the server
+   * @param port The port to use on the server
+   * @param findPort whether the server should start at the given port and 
+   *        increment by 1 until it finds a free port.
+   */
+  public HttpServer(String name, String bindAddress, int port, boolean findPort
+      ) throws IOException {
+    webServer = new org.mortbay.jetty.Server();
+    this.findPort = findPort;
+    listener = new SocketListener();
+    listener.setPort(port);
+    listener.setHost(bindAddress);
+    webServer.addListener(listener);
+
+    final String appDir = getWebAppsPath();
+    webAppContext = webServer.addWebApplication("/", appDir + "/" + name);
+    addWebapps(appDir);
+  }
+
+  /**
+   * Add webapps and servlets.
+   * @param appDir The application directory
+   * @throws IOException
+   */
+  protected void addWebapps(final String appDir) throws IOException {
+    // set up the context for "/logs/" if "hadoop.log.dir" property is defined. 
+    String logDir = System.getProperty("hadoop.log.dir");
+    if (logDir != null) {
+      HttpContext logContext = new HttpContext();
+      logContext.setContextPath("/logs/*");
+      logContext.setResourceBase(logDir);
+      logContext.addHandler(new ResourceHandler());
+      webServer.addContext(logContext);
+    }
+
+    // set up the context for "/static/*"
+    HttpContext staticContext = new HttpContext();
+    staticContext.setContextPath("/static/*");
+    staticContext.setResourceBase(appDir + "/static");
+    staticContext.addHandler(new ResourceHandler());
+    webServer.addContext(staticContext);
+
+    // set up default servlets
+    addServlet("stacks", "/stacks", StackServlet.class);
+    addServlet("logLevel", "/logLevel", LogLevel.Servlet.class);
+  }
+
+  /**
+   * Set a value in the webapp context. These values are available to the jsp
+   * pages as "application.getAttribute(name)".
+   * @param name The name of the attribute
+   * @param value The value of the attribute
+   */
+  public void setAttribute(String name, Object value) {
+    webAppContext.setAttribute(name, value);
+  }
+
+  /**
+   * Add a servlet in the server.
+   * @param name The name of the servlet (can be passed as null)
+   * @param pathSpec The path spec for the servlet
+   * @param servletClass The servlet class
+   */
+  public <T extends HttpServlet> void addServlet(String name, String pathSpec,
+      Class<T> servletClass) {
+    try {
+      if (name == null) {
+        webAppContext.addServlet(pathSpec, servletClass.getName());
+      } else {
+        webAppContext.addServlet(name, pathSpec, servletClass.getName());
+      } 
+    } catch (ClassNotFoundException cnfe) {
+      throw new RuntimeException("Problem instantiating class", cnfe);
+    } catch (InstantiationException ie) {
+      throw new RuntimeException("Problem instantiating class", ie);
+    } catch (IllegalAccessException iae) {
+      throw new RuntimeException("Problem instantiating class", iae);
+    }
+  }
+
+  /**
+   * Get the value in the webapp context.
+   * @param name The name of the attribute
+   * @return The value of the attribute
+   */
+  public Object getAttribute(String name) {
+    return webAppContext.getAttribute(name);
+  }
+
+  /**
+   * Get the pathname to the webapps files.
+   * @return the pathname as a URL
+   * @throws IOException if 'webapps' directory cannot be found on CLASSPATH.
+   */
+  protected String getWebAppsPath() throws IOException {
+    URL url = getClass().getClassLoader().getResource("webapps");
+    if (url == null) 
+      throw new IOException("webapps not found in CLASSPATH"); 
+    return url.toString();
+  }
+
+  /**
+   * Get the port that the server is on
+   * @return the port
+   */
+  public int getPort() {
+    return listener.getPort();
+  }
+
+  public void setThreads(int min, int max) {
+    listener.setMinThreads(min);
+    listener.setMaxThreads(max);
+  }
+
+  /**
+   * Configure an ssl listener on the server.
+   * @param addr address to listen on
+   * @param keystore location of the keystore
+   * @param storPass password for the keystore
+   * @param keyPass password for the key
+   */
+  public void addSslListener(InetSocketAddress addr, String keystore,
+      String storPass, String keyPass) throws IOException {
+    if (sslListener != null || webServer.isStarted()) {
+      throw new IOException("Failed to add ssl listener");
+    }
+    sslListener = new SslListener();
+    sslListener.setHost(addr.getHostName());
+    sslListener.setPort(addr.getPort());
+    sslListener.setKeystore(keystore);
+    sslListener.setPassword(storPass);
+    sslListener.setKeyPassword(keyPass);
+    webServer.addListener(sslListener);
+  }
+
+  /**
+   * Start the server. Does not wait for the server to start.
+   */
+  public void start() throws IOException {
+    try {
+      while (true) {
+        try {
+          webServer.start();
+          break;
+        } catch (org.mortbay.util.MultiException ex) {
+          // if the multi exception contains ONLY a bind exception,
+          // then try the next port number.
+          boolean needNewPort = false;
+          if(ex.size() == 1) {
+            Exception sub = ex.getException(0);
+            if (sub instanceof java.net.BindException) {
+              if(!findPort)
+                throw sub; // java.net.BindException
+              needNewPort = true;
+            }
+          }
+          if (!needNewPort)
+            throw ex;
+          listener.setPort(listener.getPort() + 1);
+        }
+      }
+    } catch (IOException ie) {
+      throw ie;
+    } catch (Exception e) {
+      IOException ie = new IOException("Problem starting http server");
+      ie.initCause(e);
+      throw ie;
+    }
+  }
+
+  /**
+   * stop the server
+   */
+  public void stop() throws InterruptedException {
+    webServer.stop();
+  }
+
+  /**
+   * A very simple servlet to serve up a text representation of the current
+   * stack traces. It both returns the stacks to the caller and logs them.
+   * Currently the stack traces are done sequentially rather than exactly the
+   * same data.
+   */
+  public static class StackServlet extends HttpServlet {
+    private static final long serialVersionUID = -6284183679759467039L;
+
+    @Override
+    public void doGet(HttpServletRequest request, HttpServletResponse response)
+      throws ServletException, IOException {
+      
+      PrintWriter out = new PrintWriter(response.getOutputStream());
+      ReflectionUtils.printThreadInfo(out, "");
+      out.close();
+      ReflectionUtils.logThreadInfo(LOG, "jsp requested", 1);      
+    }
+  }
+}

+ 3 - 3
src/hdfs/org/apache/hadoop/hdfs/server/datanode/DataNode.java

@@ -32,7 +32,7 @@ import org.apache.hadoop.net.SocketOutputStream;
 import org.apache.hadoop.util.*;
 import org.apache.hadoop.util.DiskChecker.DiskErrorException;
 import org.apache.hadoop.util.DiskChecker.DiskOutOfSpaceException;
-import org.apache.hadoop.mapred.StatusHttpServer;
+import org.apache.hadoop.http.HttpServer;
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.protocol.BlockListAsLongs;
 import org.apache.hadoop.hdfs.protocol.DatanodeID;
@@ -143,7 +143,7 @@ public class DataNode extends Configured
   long lastHeartbeat = 0;
   long heartBeatInterval;
   private DataStorage storage = null;
-  private StatusHttpServer infoServer = null;
+  private HttpServer infoServer = null;
   private DataNodeMetrics myMetrics;
   private static InetSocketAddress nameNodeAddr;
   private InetSocketAddress selfAddr;
@@ -349,7 +349,7 @@ public class DataNode extends Configured
     InetSocketAddress infoSocAddr = NetUtils.createSocketAddr(infoAddr);
     String infoHost = infoSocAddr.getHostName();
     int tmpInfoPort = infoSocAddr.getPort();
-    this.infoServer = new StatusHttpServer("datanode", infoHost, tmpInfoPort, tmpInfoPort == 0);
+    this.infoServer = new HttpServer("datanode", infoHost, tmpInfoPort, tmpInfoPort == 0);
     InetSocketAddress secInfoSocAddr = NetUtils.createSocketAddr(
         conf.get("dfs.datanode.https.address", infoHost + ":" + 0));
     Configuration sslConf = new Configuration(conf);

+ 3 - 3
src/hdfs/org/apache/hadoop/hdfs/server/namenode/FSNamesystem.java

@@ -32,7 +32,7 @@ import org.apache.hadoop.hdfs.server.namenode.metrics.FSNamesystemMetrics;
 import org.apache.hadoop.security.UnixUserGroupInformation;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.util.*;
-import org.apache.hadoop.mapred.StatusHttpServer;
+import org.apache.hadoop.http.HttpServer;
 import org.apache.hadoop.metrics.util.MBeanUtil;
 import org.apache.hadoop.net.DNSToSwitchMapping;
 import org.apache.hadoop.net.NetUtils;
@@ -187,7 +187,7 @@ public class FSNamesystem implements FSConstants, FSNamesystemMBean {
   //
   // For the HTTP browsing interface
   //
-  StatusHttpServer infoServer;
+  HttpServer infoServer;
   int infoPort;
   Date startTime;
     
@@ -339,7 +339,7 @@ public class FSNamesystem implements FSConstants, FSNamesystemMBean {
     InetSocketAddress infoSocAddr = NetUtils.createSocketAddr(infoAddr);
     String infoHost = infoSocAddr.getHostName();
     int tmpInfoPort = infoSocAddr.getPort();
-    this.infoServer = new StatusHttpServer("dfs", infoHost, tmpInfoPort, 
+    this.infoServer = new HttpServer("dfs", infoHost, tmpInfoPort, 
                                             tmpInfoPort == 0);
     InetSocketAddress secInfoSocAddr = NetUtils.createSocketAddr(
         conf.get("dfs.https.address", infoHost + ":" + 0));

+ 3 - 3
src/hdfs/org/apache/hadoop/hdfs/server/namenode/SecondaryNameNode.java

@@ -27,7 +27,7 @@ import org.apache.hadoop.ipc.*;
 import org.apache.hadoop.conf.*;
 import org.apache.hadoop.util.StringUtils;
 import org.apache.hadoop.util.Daemon;
-import org.apache.hadoop.mapred.StatusHttpServer;
+import org.apache.hadoop.http.HttpServer;
 import org.apache.hadoop.net.NetUtils;
 
 import java.io.*;
@@ -61,7 +61,7 @@ public class SecondaryNameNode implements FSConstants, Runnable {
   private Configuration conf;
   private InetSocketAddress nameNodeAddr;
   private boolean shouldRun;
-  private StatusHttpServer infoServer;
+  private HttpServer infoServer;
   private int infoPort;
   private String infoBindAddress;
 
@@ -151,7 +151,7 @@ public class SecondaryNameNode implements FSConstants, Runnable {
     InetSocketAddress infoSocAddr = NetUtils.createSocketAddr(infoAddr);
     infoBindAddress = infoSocAddr.getHostName();
     int tmpInfoPort = infoSocAddr.getPort();
-    infoServer = new StatusHttpServer("secondary", infoBindAddress, tmpInfoPort, 
+    infoServer = new HttpServer("secondary", infoBindAddress, tmpInfoPort, 
                                       tmpInfoPort == 0);
     infoServer.setAttribute("name.system.image", checkpointImage);
     this.infoServer.setAttribute("name.conf", conf);

+ 4 - 221
src/mapred/org/apache/hadoop/mapred/StatusHttpServer.java

@@ -18,42 +18,17 @@
 package org.apache.hadoop.mapred;
 
 import java.io.IOException;
-import java.io.OutputStream;
 import java.io.PrintWriter;
-import java.net.URL;
-import java.net.InetSocketAddress;
 
 import javax.servlet.ServletException;
 import javax.servlet.http.HttpServlet;
 import javax.servlet.http.HttpServletRequest;
 import javax.servlet.http.HttpServletResponse;
 
-import org.apache.commons.logging.Log;
-import org.apache.commons.logging.LogFactory;
-import org.apache.hadoop.util.ReflectionUtils;
-import org.mortbay.http.HttpContext;
-import org.mortbay.http.SocketListener;
-import org.mortbay.http.SslListener;
-import org.mortbay.http.handler.ResourceHandler;
-import org.mortbay.jetty.servlet.WebApplicationContext;
-
 /**
- * Create a Jetty embedded server to answer http requests. The primary goal
- * is to serve up status information for the server.
- * There are three contexts:
- *   "/logs/" -> points to the log directory
- *   "/static/" -> points to common static files (src/webapps/static)
- *   "/" -> the jsp server code from (src/webapps/<name>)
+ * A mapred http server.  
  */
-public class StatusHttpServer {
-  private org.mortbay.jetty.Server webServer;
-  private SocketListener listener;
-  private SslListener sslListener;
-  private boolean findPort;
-  private WebApplicationContext webAppContext;
-  private static final Log LOG =
-    LogFactory.getLog(StatusHttpServer.class.getName());
-
+public class StatusHttpServer extends org.apache.hadoop.http.HttpServer {
   /**
    * Create a status server on the given port.
    * The jsp scripts are taken from src/webapps/<name>.
@@ -62,202 +37,10 @@ public class StatusHttpServer {
    * @param findPort whether the server should start at the given port and 
    *        increment by 1 until it finds a free port.
    */
-  public StatusHttpServer(String name, String bindAddress, int port, 
+  StatusHttpServer(String name, String bindAddress, int port, 
                           boolean findPort) throws IOException {
-    webServer = new org.mortbay.jetty.Server();
-    this.findPort = findPort;
-    listener = new SocketListener();
-    listener.setPort(port);
-    listener.setHost(bindAddress);
-    webServer.addListener(listener);
-
-    // set up the context for "/logs/" if "hadoop.log.dir" property is defined. 
-    String logDir = System.getProperty("hadoop.log.dir");
-    if (logDir != null) {
-      HttpContext logContext = new HttpContext();
-      logContext.setContextPath("/logs/*");
-      logContext.setResourceBase(logDir);
-      logContext.addHandler(new ResourceHandler());
-      webServer.addContext(logContext);
-    }
-
-    // set up the context for "/static/*"
-    String appDir = getWebAppsPath();
-    HttpContext staticContext = new HttpContext();
-    staticContext.setContextPath("/static/*");
-    staticContext.setResourceBase(appDir + "/static");
-    staticContext.addHandler(new ResourceHandler());
-    webServer.addContext(staticContext);
-
-    // set up the context for "/" jsp files
-    webAppContext = 
-      webServer.addWebApplication("/", appDir + "/" + name);
-    addServlet("stacks", "/stacks", StackServlet.class);
+    super(name, bindAddress, port, findPort);
     addServlet("reducegraph", "/taskgraph", TaskGraphServlet.class);
-    addServlet("logLevel", "/logLevel",
-        org.apache.hadoop.log.LogLevel.Servlet.class);
-  }
-
-  /**
-   * Set a value in the webapp context. These values are available to the jsp
-   * pages as "application.getAttribute(name)".
-   * @param name The name of the attribute
-   * @param value The value of the attribute
-   */
-  public void setAttribute(String name, Object value) {
-    webAppContext.setAttribute(name, value);
-  }
-
-  /**
-   * Add a servlet in the server.
-   * @param name The name of the servlet (can be passed as null)
-   * @param pathSpec The path spec for the servlet
-   * @param servletClass The servlet class
-   */
-  public <T extends HttpServlet> 
-   void addServlet(String name, String pathSpec, 
-                   Class<T> servletClass) {
-   
-    WebApplicationContext context = webAppContext;
-    try {
-      if (name == null) {
-        context.addServlet(pathSpec, servletClass.getName());
-      } else {
-        context.addServlet(name, pathSpec, servletClass.getName());
-      } 
-    } catch (ClassNotFoundException ex) {
-      throw makeRuntimeException("Problem instantiating class", ex);
-    } catch (InstantiationException ex) {
-      throw makeRuntimeException("Problem instantiating class", ex);
-    } catch (IllegalAccessException ex) {
-      throw makeRuntimeException("Problem instantiating class", ex);
-    }
-  }
-
-  private static RuntimeException makeRuntimeException(String msg, 
-                                                       Throwable cause) {
-    RuntimeException result = new RuntimeException(msg);
-    if (cause != null) {
-      result.initCause(cause);
-    }
-    return result;
-  }
-
-  /**
-   * Get the value in the webapp context.
-   * @param name The name of the attribute
-   * @return The value of the attribute
-   */
-  public Object getAttribute(String name) {
-    return webAppContext.getAttribute(name);
-  }
-
-  /**
-   * Get the pathname to the webapps files.
-   * @return the pathname as a URL
-   */
-  private static String getWebAppsPath() throws IOException {
-    URL url = StatusHttpServer.class.getClassLoader().getResource("webapps");
-    if (url == null) 
-      throw new IOException("webapps not found in CLASSPATH"); 
-    return url.toString();
-  }
-
-  /**
-   * Get the port that the server is on
-   * @return the port
-   */
-  public int getPort() {
-    return listener.getPort();
-  }
-
-  public void setThreads(int min, int max) {
-    listener.setMinThreads(min);
-    listener.setMaxThreads(max);
-  }
-
-  /**
-   * Configure an ssl listener on the server.
-   * @param addr address to listen on
-   * @param keystore location of the keystore
-   * @param storPass password for the keystore
-   * @param keyPass password for the key
-   */
-  public void addSslListener(InetSocketAddress addr, String keystore,
-      String storPass, String keyPass) throws IOException {
-    if (sslListener != null || webServer.isStarted()) {
-      throw new IOException("Failed to add ssl listener");
-    }
-    sslListener = new SslListener();
-    sslListener.setHost(addr.getHostName());
-    sslListener.setPort(addr.getPort());
-    sslListener.setKeystore(keystore);
-    sslListener.setPassword(storPass);
-    sslListener.setKeyPassword(keyPass);
-    webServer.addListener(sslListener);
-  }
-
-  /**
-   * Start the server. Does not wait for the server to start.
-   */
-  public void start() throws IOException {
-    try {
-      while (true) {
-        try {
-          webServer.start();
-          break;
-        } catch (org.mortbay.util.MultiException ex) {
-          // if the multi exception contains ONLY a bind exception,
-          // then try the next port number.
-          boolean needNewPort = false;
-          if(ex.size() == 1) {
-            Exception sub = ex.getException(0);
-            if (sub instanceof java.net.BindException) {
-              if(!findPort)
-                throw sub; // java.net.BindException
-              needNewPort = true;
-            }
-          }
-          if (!needNewPort)
-            throw ex;
-          listener.setPort(listener.getPort() + 1);
-        }
-      }
-    } catch (IOException ie) {
-      throw ie;
-    } catch (Exception e) {
-      IOException ie = new IOException("Problem starting http server");
-      ie.initCause(e);
-      throw ie;
-    }
-  }
-
-  /**
-   * stop the server
-   */
-  public void stop() throws InterruptedException {
-    webServer.stop();
-  }
-
-  /**
-   * A very simple servlet to serve up a text representation of the current
-   * stack traces. It both returns the stacks to the caller and logs them.
-   * Currently the stack traces are done sequentially rather than exactly the
-   * same data.
-   */
-  public static class StackServlet extends HttpServlet {
-
-    private static final long serialVersionUID = -6284183679759467039L;
-
-    @Override
-    public void doGet(HttpServletRequest request, HttpServletResponse response)
-      throws ServletException, IOException {
-      
-      OutputStream outStream = response.getOutputStream();
-      ReflectionUtils.printThreadInfo(new PrintWriter(outStream), "");
-      outStream.close();
-      ReflectionUtils.logThreadInfo(LOG, "jsp requested", 1);      
-    }
   }
 
   /** The servlet that outputs svg graphics for map / reduce task 

+ 2 - 3
src/test/org/apache/hadoop/log/TestLogLevel.java

@@ -20,7 +20,7 @@ package org.apache.hadoop.log;
 import java.io.*;
 import java.net.*;
 
-import org.apache.hadoop.mapred.StatusHttpServer;
+import org.apache.hadoop.http.HttpServer;
 
 import junit.framework.TestCase;
 import org.apache.commons.logging.*;
@@ -42,8 +42,7 @@ public class TestLogLevel extends TestCase {
       log.error("log.error1");
       assertTrue(!Level.ERROR.equals(log.getEffectiveLevel()));
 
-      StatusHttpServer server = new StatusHttpServer("..",
-          "localhost", 22222, true);
+      HttpServer server = new HttpServer("..", "localhost", 22222, true);
       server.start();
       int port = server.getPort();