Browse Source

Revert HADOOP-8193 from r1304967. Patch introduced some NPEs in a test case.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1305152 13f79535-47bb-0310-9956-ffa450edef68
Todd Lipcon 13 years ago
parent
commit
39775dca68
18 changed files with 348 additions and 524 deletions
  1. 0 3
      hadoop-common-project/hadoop-common/CHANGES.txt
  2. 1 6
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/BadFencingConfigurationException.java
  3. 28 22
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/FailoverController.java
  4. 3 1
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/FenceMethod.java
  5. 41 11
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/HAAdmin.java
  6. 0 74
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/HAServiceTarget.java
  7. 3 2
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/NodeFencer.java
  8. 1 2
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/ShellCommandFencer.java
  9. 1 2
      hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/SshFenceByTcpPort.java
  10. 0 94
      hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/DummyHAService.java
  11. 214 144
      hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestFailoverController.java
  12. 6 6
      hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestHAAdmin.java
  13. 19 28
      hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestNodeFencer.java
  14. 12 11
      hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestShellCommandFencer.java
  15. 7 19
      hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestSshFenceByTcpPort.java
  16. 9 3
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/tools/DFSHAAdmin.java
  17. 0 84
      hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/tools/NNHAServiceTarget.java
  18. 3 12
      hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/tools/TestDFSHAAdmin.java

+ 0 - 3
hadoop-common-project/hadoop-common/CHANGES.txt

@@ -220,9 +220,6 @@ Release 0.23.3 - UNRELEASED
     HADOOP-8163. Improve ActiveStandbyElector to provide hooks for
     fencing old active. (todd)
 
-    HADOOP-8193. Refactor FailoverController/HAAdmin code to add an abstract
-    class for "target" services. (todd)
-
   OPTIMIZATIONS
 
   BUG FIXES

+ 1 - 6
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/BadFencingConfigurationException.java

@@ -19,16 +19,11 @@ package org.apache.hadoop.ha;
 
 import java.io.IOException;
 
-import org.apache.hadoop.classification.InterfaceAudience;
-import org.apache.hadoop.classification.InterfaceStability;
-
 /**
  * Indicates that the operator has specified an invalid configuration
  * for fencing methods.
  */
-@InterfaceAudience.Public
-@InterfaceStability.Evolving
-public class BadFencingConfigurationException extends IOException {
+class BadFencingConfigurationException extends IOException {
   private static final long serialVersionUID = 1L;
 
   public BadFencingConfigurationException(String msg) {

+ 28 - 22
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/FailoverController.java

@@ -18,6 +18,7 @@
 package org.apache.hadoop.ha;
 
 import java.io.IOException;
+import java.net.InetSocketAddress;
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
@@ -50,21 +51,21 @@ public class FailoverController {
    * allow it to become active, eg because it triggers a log roll
    * so the standby can learn about new blocks and leave safemode.
    *
-   * @param target service to make active
+   * @param toSvc service to make active
+   * @param toSvcName name of service to make active
    * @param forceActive ignore toSvc if it reports that it is not ready
    * @throws FailoverFailedException if we should avoid failover
    */
-  private static void preFailoverChecks(HAServiceTarget target,
+  private static void preFailoverChecks(HAServiceProtocol toSvc,
+                                        InetSocketAddress toSvcAddr,
                                         boolean forceActive)
       throws FailoverFailedException {
     HAServiceStatus toSvcStatus;
-    HAServiceProtocol toSvc;
 
     try {
-      toSvc = target.getProxy();
       toSvcStatus = toSvc.getServiceStatus();
     } catch (IOException e) {
-      String msg = "Unable to get service state for " + target;
+      String msg = "Unable to get service state for " + toSvcAddr;
       LOG.error(msg, e);
       throw new FailoverFailedException(msg, e);
     }
@@ -78,7 +79,7 @@ public class FailoverController {
       String notReadyReason = toSvcStatus.getNotReadyReason();
       if (!forceActive) {
         throw new FailoverFailedException(
-            target + " is not ready to become active: " +
+            toSvcAddr + " is not ready to become active: " +
             notReadyReason);
       } else {
         LOG.warn("Service is not ready to become active, but forcing: " +
@@ -102,39 +103,44 @@ public class FailoverController {
    * then try to failback.
    *
    * @param fromSvc currently active service
+   * @param fromSvcAddr addr of the currently active service
    * @param toSvc service to make active
+   * @param toSvcAddr addr of the service to make active
+   * @param fencer for fencing fromSvc
    * @param forceFence to fence fromSvc even if not strictly necessary
    * @param forceActive try to make toSvc active even if it is not ready
    * @throws FailoverFailedException if the failover fails
    */
-  public static void failover(HAServiceTarget fromSvc,
-                              HAServiceTarget toSvc,
+  public static void failover(HAServiceProtocol fromSvc,
+                              InetSocketAddress fromSvcAddr,
+                              HAServiceProtocol toSvc,
+                              InetSocketAddress toSvcAddr,
+                              NodeFencer fencer,
                               boolean forceFence,
                               boolean forceActive)
       throws FailoverFailedException {
-    Preconditions.checkArgument(fromSvc.getFencer() != null,
-        "failover requires a fencer");
-    preFailoverChecks(toSvc, forceActive);
+    Preconditions.checkArgument(fencer != null, "failover requires a fencer");
+    preFailoverChecks(toSvc, toSvcAddr, forceActive);
 
     // Try to make fromSvc standby
     boolean tryFence = true;
     try {
-      HAServiceProtocolHelper.transitionToStandby(fromSvc.getProxy());
+      HAServiceProtocolHelper.transitionToStandby(fromSvc);
       // We should try to fence if we failed or it was forced
       tryFence = forceFence ? true : false;
     } catch (ServiceFailedException sfe) {
-      LOG.warn("Unable to make " + fromSvc + " standby (" +
+      LOG.warn("Unable to make " + fromSvcAddr + " standby (" +
           sfe.getMessage() + ")");
     } catch (IOException ioe) {
-      LOG.warn("Unable to make " + fromSvc +
+      LOG.warn("Unable to make " + fromSvcAddr +
           " standby (unable to connect)", ioe);
     }
 
     // Fence fromSvc if it's required or forced by the user
     if (tryFence) {
-      if (!fromSvc.getFencer().fence(fromSvc)) {
+      if (!fencer.fence(fromSvcAddr)) {
         throw new FailoverFailedException("Unable to fence " +
-            fromSvc + ". Fencing failed.");
+            fromSvcAddr + ". Fencing failed.");
       }
     }
 
@@ -142,14 +148,14 @@ public class FailoverController {
     boolean failed = false;
     Throwable cause = null;
     try {
-      HAServiceProtocolHelper.transitionToActive(toSvc.getProxy());
+      HAServiceProtocolHelper.transitionToActive(toSvc);
     } catch (ServiceFailedException sfe) {
-      LOG.error("Unable to make " + toSvc + " active (" +
+      LOG.error("Unable to make " + toSvcAddr + " active (" +
           sfe.getMessage() + "). Failing back.");
       failed = true;
       cause = sfe;
     } catch (IOException ioe) {
-      LOG.error("Unable to make " + toSvc +
+      LOG.error("Unable to make " + toSvcAddr +
           " active (unable to connect). Failing back.", ioe);
       failed = true;
       cause = ioe;
@@ -157,7 +163,7 @@ public class FailoverController {
 
     // We failed to make toSvc active
     if (failed) {
-      String msg = "Unable to failover to " + toSvc;
+      String msg = "Unable to failover to " + toSvcAddr;
       // Only try to failback if we didn't fence fromSvc
       if (!tryFence) {
         try {
@@ -165,9 +171,9 @@ public class FailoverController {
           // become active, eg we timed out waiting for its response.
           // Unconditionally force fromSvc to become active since it
           // was previously active when we initiated failover.
-          failover(toSvc, fromSvc, true, true);
+          failover(toSvc, toSvcAddr, fromSvc, fromSvcAddr, fencer, true, true);
         } catch (FailoverFailedException ffe) {
-          msg += ". Failback to " + fromSvc +
+          msg += ". Failback to " + fromSvcAddr +
             " failed (" + ffe.getMessage() + ")";
           LOG.fatal(msg);
         }

+ 3 - 1
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/FenceMethod.java

@@ -17,6 +17,8 @@
  */
 package org.apache.hadoop.ha;
 
+import java.net.InetSocketAddress;
+
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 import org.apache.hadoop.conf.Configurable;
@@ -60,6 +62,6 @@ public interface FenceMethod {
    * @throws BadFencingConfigurationException if the configuration was
    *         determined to be invalid only at runtime
    */
-  public boolean tryFence(HAServiceTarget target, String args)
+  public boolean tryFence(InetSocketAddress serviceAddr, String args)
     throws BadFencingConfigurationException;
 }

+ 41 - 11
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/HAAdmin.java

@@ -19,6 +19,7 @@ package org.apache.hadoop.ha;
 
 import java.io.IOException;
 import java.io.PrintStream;
+import java.net.InetSocketAddress;
 import java.util.Map;
 
 import org.apache.commons.cli.Options;
@@ -27,8 +28,11 @@ import org.apache.commons.cli.CommandLineParser;
 import org.apache.commons.cli.GnuParser;
 import org.apache.commons.cli.ParseException;
 
+import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.conf.Configured;
+import org.apache.hadoop.ha.protocolPB.HAServiceProtocolClientSideTranslatorPB;
+import org.apache.hadoop.net.NetUtils;
 import org.apache.hadoop.util.Tool;
 import org.apache.hadoop.util.ToolRunner;
 
@@ -73,8 +77,6 @@ public abstract class HAAdmin extends Configured implements Tool {
   protected PrintStream errOut = System.err;
   PrintStream out = System.out;
 
-  protected abstract HAServiceTarget resolveTarget(String string);
-
   protected String getUsageString() {
     return "Usage: HAAdmin";
   }
@@ -107,7 +109,7 @@ public abstract class HAAdmin extends Configured implements Tool {
       return -1;
     }
     
-    HAServiceProtocol proto = resolveTarget(argv[1]).getProxy();
+    HAServiceProtocol proto = getProtocol(argv[1]);
     HAServiceProtocolHelper.transitionToActive(proto);
     return 0;
   }
@@ -120,13 +122,14 @@ public abstract class HAAdmin extends Configured implements Tool {
       return -1;
     }
     
-    HAServiceProtocol proto = resolveTarget(argv[1]).getProxy();
+    HAServiceProtocol proto = getProtocol(argv[1]);
     HAServiceProtocolHelper.transitionToStandby(proto);
     return 0;
   }
 
   private int failover(final String[] argv)
       throws IOException, ServiceFailedException {
+    Configuration conf = getConf();
     boolean forceFence = false;
     boolean forceActive = false;
 
@@ -159,12 +162,29 @@ public abstract class HAAdmin extends Configured implements Tool {
       return -1;
     }
 
-    HAServiceTarget fromNode = resolveTarget(args[0]);
-    HAServiceTarget toNode = resolveTarget(args[1]);
-    
+    NodeFencer fencer;
     try {
-      FailoverController.failover(fromNode, toNode,
-          forceFence, forceActive); 
+      fencer = NodeFencer.create(conf);
+    } catch (BadFencingConfigurationException bfce) {
+      errOut.println("failover: incorrect fencing configuration: " + 
+          bfce.getLocalizedMessage());
+      return -1;
+    }
+    if (fencer == null) {
+      errOut.println("failover: no fencer configured");
+      return -1;
+    }
+
+    InetSocketAddress addr1 = 
+      NetUtils.createSocketAddr(getServiceAddr(args[0]));
+    InetSocketAddress addr2 = 
+      NetUtils.createSocketAddr(getServiceAddr(args[1]));
+    HAServiceProtocol proto1 = getProtocol(args[0]);
+    HAServiceProtocol proto2 = getProtocol(args[1]);
+
+    try {
+      FailoverController.failover(proto1, addr1, proto2, addr2,
+          fencer, forceFence, forceActive); 
       out.println("Failover from "+args[0]+" to "+args[1]+" successful");
     } catch (FailoverFailedException ffe) {
       errOut.println("Failover failed: " + ffe.getLocalizedMessage());
@@ -181,7 +201,7 @@ public abstract class HAAdmin extends Configured implements Tool {
       return -1;
     }
     
-    HAServiceProtocol proto = resolveTarget(argv[1]).getProxy();
+    HAServiceProtocol proto = getProtocol(argv[1]);
     try {
       HAServiceProtocolHelper.monitorHealth(proto);
     } catch (HealthCheckFailedException e) {
@@ -199,7 +219,7 @@ public abstract class HAAdmin extends Configured implements Tool {
       return -1;
     }
 
-    HAServiceProtocol proto = resolveTarget(argv[1]).getProxy();
+    HAServiceProtocol proto = getProtocol(argv[1]);
     out.println(proto.getServiceStatus().getState());
     return 0;
   }
@@ -212,6 +232,16 @@ public abstract class HAAdmin extends Configured implements Tool {
     return serviceId;
   }
 
+  /**
+   * Return a proxy to the specified target service.
+   */
+  protected HAServiceProtocol getProtocol(String serviceId)
+      throws IOException {
+    String serviceAddr = getServiceAddr(serviceId);
+    InetSocketAddress addr = NetUtils.createSocketAddr(serviceAddr);
+    return new HAServiceProtocolClientSideTranslatorPB(addr, getConf());
+  }
+
   @Override
   public int run(String[] argv) throws Exception {
     try {

+ 0 - 74
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/HAServiceTarget.java

@@ -1,74 +0,0 @@
-/**
- * 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.ha;
-
-import java.io.IOException;
-import java.net.InetSocketAddress;
-
-import org.apache.hadoop.classification.InterfaceAudience;
-import org.apache.hadoop.classification.InterfaceStability;
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
-import org.apache.hadoop.ha.protocolPB.HAServiceProtocolClientSideTranslatorPB;
-
-/**
- * Represents a target of the client side HA administration commands.
- */
-@InterfaceAudience.Public
-@InterfaceStability.Evolving
-public abstract class HAServiceTarget {
-
-  /**
-   * @return the IPC address of the target node.
-   */
-  public abstract InetSocketAddress getAddress();
-
-  /**
-   * @return a Fencer implementation configured for this target node
-   */
-  public abstract NodeFencer getFencer();
-  
-  /**
-   * @throws BadFencingConfigurationException if the fencing configuration
-   * appears to be invalid. This is divorced from the above
-   * {@link #getFencer()} method so that the configuration can be checked
-   * during the pre-flight phase of failover.
-   */
-  public abstract void checkFencingConfigured()
-      throws BadFencingConfigurationException;
-  
-  /**
-   * @return a proxy to connect to the target HA Service.
-   */
-  public HAServiceProtocol getProxy(Configuration conf, int timeoutMs)
-      throws IOException {
-    Configuration confCopy = new Configuration(conf);
-    // Lower the timeout so we quickly fail to connect
-    confCopy.setInt(CommonConfigurationKeysPublic.IPC_CLIENT_CONNECT_MAX_RETRIES_KEY, 1);
-    return new HAServiceProtocolClientSideTranslatorPB(
-        getAddress(),
-        confCopy, null, timeoutMs);
-  }
-
-  /**
-   * @return a proxy to connect to the target HA Service.
-   */
-  public final HAServiceProtocol getProxy() throws IOException {
-    return getProxy(new Configuration(), 0); // default conf, timeout
-  }
-}

+ 3 - 2
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/NodeFencer.java

@@ -17,6 +17,7 @@
  */
 package org.apache.hadoop.ha;
 
+import java.net.InetSocketAddress;
 import java.util.List;
 import java.util.Map;
 import java.util.regex.Matcher;
@@ -90,14 +91,14 @@ public class NodeFencer {
     return new NodeFencer(conf);
   }
 
-  public boolean fence(HAServiceTarget fromSvc) {
+  public boolean fence(InetSocketAddress serviceAddr) {
     LOG.info("====== Beginning Service Fencing Process... ======");
     int i = 0;
     for (FenceMethodWithArg method : methods) {
       LOG.info("Trying method " + (++i) + "/" + methods.size() +": " + method);
       
       try {
-        if (method.method.tryFence(fromSvc, method.arg)) {
+        if (method.method.tryFence(serviceAddr, method.arg)) {
           LOG.info("====== Fencing successful by method " + method + " ======");
           return true;
         }

+ 1 - 2
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/ShellCommandFencer.java

@@ -75,8 +75,7 @@ public class ShellCommandFencer
   }
 
   @Override
-  public boolean tryFence(HAServiceTarget target, String cmd) {
-    InetSocketAddress serviceAddr = target.getAddress();
+  public boolean tryFence(InetSocketAddress serviceAddr, String cmd) {
     List<String> cmdList = Arrays.asList(cmd.split("\\s+"));
 
     // Create arg list with service as the first argument

+ 1 - 2
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/ha/SshFenceByTcpPort.java

@@ -79,11 +79,10 @@ public class SshFenceByTcpPort extends Configured
   }
 
   @Override
-  public boolean tryFence(HAServiceTarget target, String argsStr)
+  public boolean tryFence(InetSocketAddress serviceAddr, String argsStr)
       throws BadFencingConfigurationException {
 
     Args args = new Args(argsStr);
-    InetSocketAddress serviceAddr = target.getAddress();
     String host = serviceAddr.getHostName();
     
     Session session;

+ 0 - 94
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/DummyHAService.java

@@ -1,94 +0,0 @@
-/**
- * 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.ha;
-
-import java.io.IOException;
-import java.net.InetSocketAddress;
-
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
-import org.apache.hadoop.security.AccessControlException;
-import org.mockito.Mockito;
-
-/**
- * Test-only implementation of {@link HAServiceTarget}, which returns
- * a mock implementation.
- */
-class DummyHAService extends HAServiceTarget {
-  HAServiceState state;
-  HAServiceProtocol proxy;
-  NodeFencer fencer;
-  InetSocketAddress address;
-
-  DummyHAService(HAServiceState state, InetSocketAddress address) {
-    this.state = state;
-    this.proxy = makeMock();
-    this.fencer = Mockito.mock(NodeFencer.class);
-    this.address = address;
-  }
-  
-  private HAServiceProtocol makeMock() {
-    return Mockito.spy(new HAServiceProtocol() {
-      @Override
-      public void monitorHealth() throws HealthCheckFailedException,
-          AccessControlException, IOException {
-      }
-
-      @Override
-      public void transitionToActive() throws ServiceFailedException,
-          AccessControlException, IOException {
-        state = HAServiceState.ACTIVE;
-      }
-
-      @Override
-      public void transitionToStandby() throws ServiceFailedException,
-          AccessControlException, IOException {
-        state = HAServiceState.STANDBY;
-      }
-
-      @Override
-      public HAServiceStatus getServiceStatus() throws IOException {
-        HAServiceStatus ret = new HAServiceStatus(state);
-        if (state == HAServiceState.STANDBY) {
-          ret.setReadyToBecomeActive();
-        }
-        return ret;
-      }
-    });
-  }
-
-  @Override
-  public InetSocketAddress getAddress() {
-    return address;
-  }
-
-  @Override
-  public HAServiceProtocol getProxy(Configuration conf, int timeout)
-      throws IOException {
-    return proxy;
-  }
-
-  @Override
-  public NodeFencer getFencer() {
-    return fencer;
-  }
-
-  @Override
-  public void checkFencingConfigured() throws BadFencingConfigurationException {
-  }
-}

+ 214 - 144
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestFailoverController.java

@@ -24,85 +24,124 @@ import static org.mockito.Mockito.spy;
 import static org.mockito.Mockito.verify;
 
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
 import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
+import org.apache.hadoop.ha.protocolPB.HAServiceProtocolClientSideTranslatorPB;
 import org.apache.hadoop.ha.TestNodeFencer.AlwaysSucceedFencer;
 import org.apache.hadoop.ha.TestNodeFencer.AlwaysFailFencer;
 import static org.apache.hadoop.ha.TestNodeFencer.setupFencer;
+import org.apache.hadoop.net.NetUtils;
 import org.apache.hadoop.security.AccessControlException;
 
 import org.junit.Test;
-import org.mockito.Mockito;
-import org.mockito.internal.stubbing.answers.ThrowsException;
-import org.mockito.stubbing.Answer;
-
 import static org.junit.Assert.*;
 
 public class TestFailoverController {
+
   private InetSocketAddress svc1Addr = new InetSocketAddress("svc1", 1234); 
-  private InetSocketAddress svc2Addr = new InetSocketAddress("svc2", 5678);
+  private InetSocketAddress svc2Addr = new InetSocketAddress("svc2", 5678); 
+
+  private class DummyService implements HAServiceProtocol {
+    HAServiceState state;
+
+    DummyService(HAServiceState state) {
+      this.state = state;
+    }
+
+    @Override
+    public void monitorHealth() throws HealthCheckFailedException, IOException {
+      // Do nothing
+    }
+
+    @Override
+    public void transitionToActive() throws ServiceFailedException, IOException {
+      state = HAServiceState.ACTIVE;
+    }
 
-  HAServiceStatus STATE_NOT_READY = new HAServiceStatus(HAServiceState.STANDBY)
-      .setNotReadyToBecomeActive("injected not ready");
+    @Override
+    public void transitionToStandby() throws ServiceFailedException, IOException {
+      state = HAServiceState.STANDBY;
+    }
 
+    @Override
+    public HAServiceStatus getServiceStatus() throws IOException {
+      HAServiceStatus ret = new HAServiceStatus(state);
+      if (state == HAServiceState.STANDBY) {
+        ret.setReadyToBecomeActive();
+      }
+      return ret;
+    }
+    
+    private HAServiceState getServiceState() {
+      return state;
+    }
+  }
+  
   @Test
   public void testFailoverAndFailback() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY);
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     AlwaysSucceedFencer.fenceCalled = 0;
-    FailoverController.failover(svc1, svc2, false, false);
+    FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
     assertEquals(0, TestNodeFencer.AlwaysSucceedFencer.fenceCalled);
-    assertEquals(HAServiceState.STANDBY, svc1.state);
-    assertEquals(HAServiceState.ACTIVE, svc2.state);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
+    assertEquals(HAServiceState.ACTIVE, svc2.getServiceState());
 
     AlwaysSucceedFencer.fenceCalled = 0;
-    FailoverController.failover(svc2, svc1, false, false);
+    FailoverController.failover(svc2, svc2Addr, svc1, svc1Addr, fencer, false, false);
     assertEquals(0, TestNodeFencer.AlwaysSucceedFencer.fenceCalled);
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverFromStandbyToStandby() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.STANDBY, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.STANDBY);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY);
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
-    FailoverController.failover(svc1, svc2, false, false);
-    assertEquals(HAServiceState.STANDBY, svc1.state);
-    assertEquals(HAServiceState.ACTIVE, svc2.state);
+    FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
+    assertEquals(HAServiceState.ACTIVE, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverFromActiveToActive() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.ACTIVE, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.ACTIVE);
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Can't failover to an already active service");
     } catch (FailoverFailedException ffe) {
       // Expected
     }
 
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.ACTIVE, svc2.state);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.ACTIVE, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverWithoutPermission() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    Mockito.doThrow(new AccessControlException("Access denied"))
-        .when(svc1.proxy).getServiceStatus();
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new AccessControlException("Access denied"))
-        .when(svc2.proxy).getServiceStatus();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE) {
+      @Override
+      public HAServiceStatus getServiceStatus() throws IOException {
+        throw new AccessControlException("Access denied");
+      }
+    };
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public HAServiceStatus getServiceStatus() throws IOException {
+        throw new AccessControlException("Access denied");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Can't failover when access is denied");
     } catch (FailoverFailedException ffe) {
       assertTrue(ffe.getCause().getMessage().contains("Access denied"));
@@ -112,13 +151,19 @@ public class TestFailoverController {
 
   @Test
   public void testFailoverToUnreadyService() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doReturn(STATE_NOT_READY).when(svc2.proxy).getServiceStatus();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public HAServiceStatus getServiceStatus() throws IOException {
+        HAServiceStatus ret = new HAServiceStatus(HAServiceState.STANDBY);
+        ret.setNotReadyToBecomeActive("injected not ready");
+        return ret;
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Can't failover to a service that's not ready");
     } catch (FailoverFailedException ffe) {
       // Expected
@@ -127,88 +172,95 @@ public class TestFailoverController {
       }
     }
 
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
 
     // Forcing it means we ignore readyToBecomeActive
-    FailoverController.failover(svc1, svc2, false, true);
-    assertEquals(HAServiceState.STANDBY, svc1.state);
-    assertEquals(HAServiceState.ACTIVE, svc2.state);
+    FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, true);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
+    assertEquals(HAServiceState.ACTIVE, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverToUnhealthyServiceFailsAndFailsback() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new HealthCheckFailedException("Failed!"))
-        .when(svc2.proxy).monitorHealth();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public void monitorHealth() throws HealthCheckFailedException {
+        throw new HealthCheckFailedException("Failed!");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Failover to unhealthy service");
     } catch (FailoverFailedException ffe) {
       // Expected
     }
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverFromFaultyServiceSucceeds() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc1.proxy).transitionToStandby();
-
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE) {
+      @Override
+      public void transitionToStandby() throws ServiceFailedException {
+        throw new ServiceFailedException("Failed!");
+      }
+    };
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY);
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     AlwaysSucceedFencer.fenceCalled = 0;
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
     } catch (FailoverFailedException ffe) {
       fail("Faulty active prevented failover");
     }
 
     // svc1 still thinks it's active, that's OK, it was fenced
     assertEquals(1, AlwaysSucceedFencer.fenceCalled);
-    assertSame(svc1, AlwaysSucceedFencer.fencedSvc);
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.ACTIVE, svc2.state);
+    assertEquals("svc1:1234", AlwaysSucceedFencer.fencedSvc);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.ACTIVE, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverFromFaultyServiceFencingFailure() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc1.proxy).transitionToStandby();
-
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysFailFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE) {
+      @Override
+      public void transitionToStandby() throws ServiceFailedException {
+        throw new ServiceFailedException("Failed!");
+      }
+    };
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY);
+    NodeFencer fencer = setupFencer(AlwaysFailFencer.class.getName());
 
     AlwaysFailFencer.fenceCalled = 0;
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Failed over even though fencing failed");
     } catch (FailoverFailedException ffe) {
       // Expected
     }
 
     assertEquals(1, AlwaysFailFencer.fenceCalled);
-    assertSame(svc1, AlwaysFailFencer.fencedSvc);
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals("svc1:1234", AlwaysFailFencer.fencedSvc);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
 
   @Test
   public void testFencingFailureDuringFailover() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysFailFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY);
+    NodeFencer fencer = setupFencer(AlwaysFailFencer.class.getName());
 
     AlwaysFailFencer.fenceCalled = 0;
     try {
-      FailoverController.failover(svc1, svc2, true, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, true, false);
       fail("Failed over even though fencing requested and failed");
     } catch (FailoverFailedException ffe) {
       // Expected
@@ -217,83 +269,90 @@ public class TestFailoverController {
     // If fencing was requested and it failed we don't try to make
     // svc2 active anyway, and we don't failback to svc1.
     assertEquals(1, AlwaysFailFencer.fenceCalled);
-    assertSame(svc1, AlwaysFailFencer.fencedSvc);
-    assertEquals(HAServiceState.STANDBY, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals("svc1:1234", AlwaysFailFencer.fencedSvc);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
   
+  private HAServiceProtocol getProtocol(String target)
+      throws IOException {
+    InetSocketAddress addr = NetUtils.createSocketAddr(target);
+    Configuration conf = new Configuration();
+    // Lower the timeout so we quickly fail to connect
+    conf.setInt(CommonConfigurationKeysPublic.IPC_CLIENT_CONNECT_MAX_RETRIES_KEY, 1);
+    return new HAServiceProtocolClientSideTranslatorPB(addr, conf);
+  }
+
   @Test
   public void testFailoverFromNonExistantServiceWithFencer() throws Exception {
-    DummyHAService svc1 = spy(new DummyHAService(null, svc1Addr));
-    // Getting a proxy to a dead server will throw IOException on call,
-    // not on creation of the proxy.
-    HAServiceProtocol errorThrowingProxy = Mockito.mock(HAServiceProtocol.class,
-        new ThrowsException(new IOException("Could not connect to host")));
-    Mockito.doReturn(errorThrowingProxy).when(svc1).getProxy();
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    HAServiceProtocol svc1 = getProtocol("localhost:1234");
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY);
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
     } catch (FailoverFailedException ffe) {
       fail("Non-existant active prevented failover");
     }
 
     // Don't check svc1 because we can't reach it, but that's OK, it's been fenced.
-    assertEquals(HAServiceState.ACTIVE, svc2.state);
+    assertEquals(HAServiceState.ACTIVE, svc2.getServiceState());
   }
 
   @Test
   public void testFailoverToNonExistantServiceFails() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = spy(new DummyHAService(null, svc2Addr));
-    Mockito.doThrow(new IOException("Failed to connect"))
-      .when(svc2).getProxy(Mockito.<Configuration>any(),
-          Mockito.anyInt());
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    HAServiceProtocol svc2 = getProtocol("localhost:1234");
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Failed over to a non-existant standby");
     } catch (FailoverFailedException ffe) {
       // Expected
     }
 
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
   }
 
   @Test
   public void testFailoverToFaultyServiceFailsbackOK() throws Exception {
-    DummyHAService svc1 = spy(new DummyHAService(HAServiceState.ACTIVE, svc1Addr));
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc2.proxy).transitionToActive();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = spy(new DummyService(HAServiceState.ACTIVE));
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public void transitionToActive() throws ServiceFailedException {
+        throw new ServiceFailedException("Failed!");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Failover to already active service");
     } catch (FailoverFailedException ffe) {
       // Expected
     }
 
     // svc1 went standby then back to active
-    verify(svc1.proxy).transitionToStandby();
-    verify(svc1.proxy).transitionToActive();
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    verify(svc1).transitionToStandby();
+    verify(svc1).transitionToActive();
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
 
   @Test
   public void testWeDontFailbackIfActiveWasFenced() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc2.proxy).transitionToActive();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public void transitionToActive() throws ServiceFailedException {
+        throw new ServiceFailedException("Failed!");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, true, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, true, false);
       fail("Failed over to service that won't transition to active");
     } catch (FailoverFailedException ffe) {
       // Expected
@@ -301,21 +360,24 @@ public class TestFailoverController {
 
     // We failed to failover and did not failback because we fenced
     // svc1 (we forced it), therefore svc1 and svc2 should be standby.
-    assertEquals(HAServiceState.STANDBY, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
 
   @Test
   public void testWeFenceOnFailbackIfTransitionToActiveFails() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc2.proxy).transitionToActive();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public void transitionToActive() throws ServiceFailedException, IOException {
+        throw new IOException("Failed!");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
     AlwaysSucceedFencer.fenceCalled = 0;
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Failed over to service that won't transition to active");
     } catch (FailoverFailedException ffe) {
       // Expected
@@ -324,22 +386,25 @@ public class TestFailoverController {
     // We failed to failover. We did not fence svc1 because it cooperated
     // and we didn't force it, so we failed back to svc1 and fenced svc2.
     // Note svc2 still thinks it's active, that's OK, we fenced it.
-    assertEquals(HAServiceState.ACTIVE, svc1.state);
+    assertEquals(HAServiceState.ACTIVE, svc1.getServiceState());
     assertEquals(1, AlwaysSucceedFencer.fenceCalled);
-    assertSame(svc2, AlwaysSucceedFencer.fencedSvc);
+    assertEquals("svc2:5678", AlwaysSucceedFencer.fencedSvc);
   }
 
   @Test
   public void testFailureToFenceOnFailbackFailsTheFailback() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new IOException("Failed!"))
-        .when(svc2.proxy).transitionToActive();
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysFailFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE);
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public void transitionToActive() throws ServiceFailedException, IOException {
+        throw new IOException("Failed!");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysFailFencer.class.getName());
     AlwaysFailFencer.fenceCalled = 0;
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1,  svc1Addr,  svc2,  svc2Addr, fencer, false, false);
       fail("Failed over to service that won't transition to active");
     } catch (FailoverFailedException ffe) {
       // Expected
@@ -348,30 +413,35 @@ public class TestFailoverController {
     // We did not fence svc1 because it cooperated and we didn't force it, 
     // we failed to failover so we fenced svc2, we failed to fence svc2
     // so we did not failback to svc1, ie it's still standby.
-    assertEquals(HAServiceState.STANDBY, svc1.state);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
     assertEquals(1, AlwaysFailFencer.fenceCalled);
-    assertSame(svc2, AlwaysFailFencer.fencedSvc);
+    assertEquals("svc2:5678", AlwaysFailFencer.fencedSvc);
   }
 
   @Test
   public void testFailbackToFaultyServiceFails() throws Exception {
-    DummyHAService svc1 = new DummyHAService(HAServiceState.ACTIVE, svc1Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc1.proxy).transitionToActive();
-    DummyHAService svc2 = new DummyHAService(HAServiceState.STANDBY, svc2Addr);
-    Mockito.doThrow(new ServiceFailedException("Failed!"))
-        .when(svc2.proxy).transitionToActive();
-
-    svc1.fencer = svc2.fencer = setupFencer(AlwaysSucceedFencer.class.getName());
+    DummyService svc1 = new DummyService(HAServiceState.ACTIVE) {
+      @Override
+      public void transitionToActive() throws ServiceFailedException {
+        throw new ServiceFailedException("Failed!");
+      }
+    };
+    DummyService svc2 = new DummyService(HAServiceState.STANDBY) {
+      @Override
+      public void transitionToActive() throws ServiceFailedException {
+        throw new ServiceFailedException("Failed!");
+      }
+    };
+    NodeFencer fencer = setupFencer(AlwaysSucceedFencer.class.getName());
 
     try {
-      FailoverController.failover(svc1, svc2, false, false);
+      FailoverController.failover(svc1, svc1Addr, svc2, svc2Addr, fencer, false, false);
       fail("Failover to already active service");
     } catch (FailoverFailedException ffe) {
       // Expected
     }
 
-    assertEquals(HAServiceState.STANDBY, svc1.state);
-    assertEquals(HAServiceState.STANDBY, svc2.state);
+    assertEquals(HAServiceState.STANDBY, svc1.getServiceState());
+    assertEquals(HAServiceState.STANDBY, svc2.getServiceState());
   }
 }

+ 6 - 6
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestHAAdmin.java

@@ -22,15 +22,14 @@ import static org.junit.Assert.*;
 import java.io.ByteArrayOutputStream;
 import java.io.IOException;
 import java.io.PrintStream;
-import java.net.InetSocketAddress;
 
 import org.apache.commons.logging.LogFactory;
 import org.apache.commons.logging.Log;
 import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
 
 import org.junit.Before;
 import org.junit.Test;
+import org.mockito.Mockito;
 
 import com.google.common.base.Charsets;
 import com.google.common.base.Joiner;
@@ -41,14 +40,15 @@ public class TestHAAdmin {
   private HAAdmin tool;
   private ByteArrayOutputStream errOutBytes = new ByteArrayOutputStream();
   private String errOutput;
-
+  private HAServiceProtocol mockProtocol;
+  
   @Before
   public void setup() throws IOException {
+    mockProtocol = Mockito.mock(HAServiceProtocol.class);
     tool = new HAAdmin() {
       @Override
-      protected HAServiceTarget resolveTarget(String target) {
-        return new DummyHAService(HAServiceState.STANDBY,
-            new InetSocketAddress("dummy", 12345));
+      protected HAServiceProtocol getProtocol(String target) throws IOException {
+        return mockProtocol;
       }
     };
     tool.setConf(new Configuration());

+ 19 - 28
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestNodeFencer.java

@@ -26,35 +26,26 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.conf.Configured;
 import org.junit.Before;
 import org.junit.Test;
-import org.mockito.Mockito;
 
 import com.google.common.collect.Lists;
 
 public class TestNodeFencer {
 
-  private HAServiceTarget MOCK_TARGET;
-  
-
   @Before
   public void clearMockState() {
     AlwaysSucceedFencer.fenceCalled = 0;
     AlwaysSucceedFencer.callArgs.clear();
     AlwaysFailFencer.fenceCalled = 0;
     AlwaysFailFencer.callArgs.clear();
-    
-    MOCK_TARGET = Mockito.mock(HAServiceTarget.class);
-    Mockito.doReturn("my mock").when(MOCK_TARGET).toString();
-    Mockito.doReturn(new InetSocketAddress("host", 1234))
-        .when(MOCK_TARGET).getAddress();
   }
 
   @Test
   public void testSingleFencer() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer(
         AlwaysSucceedFencer.class.getName() + "(foo)");
-    assertTrue(fencer.fence(MOCK_TARGET));
+    assertTrue(fencer.fence(new InetSocketAddress("host", 1234)));
     assertEquals(1, AlwaysSucceedFencer.fenceCalled);
-    assertSame(MOCK_TARGET, AlwaysSucceedFencer.fencedSvc);
+    assertEquals("host:1234", AlwaysSucceedFencer.fencedSvc);
     assertEquals("foo", AlwaysSucceedFencer.callArgs.get(0));
   }
   
@@ -63,7 +54,7 @@ public class TestNodeFencer {
     NodeFencer fencer = setupFencer(
         AlwaysSucceedFencer.class.getName() + "(foo)\n" +
         AlwaysSucceedFencer.class.getName() + "(bar)\n");
-    assertTrue(fencer.fence(MOCK_TARGET));
+    assertTrue(fencer.fence(new InetSocketAddress("host", 1234)));
     // Only one call, since the first fencer succeeds
     assertEquals(1, AlwaysSucceedFencer.fenceCalled);
     assertEquals("foo", AlwaysSucceedFencer.callArgs.get(0));
@@ -77,12 +68,12 @@ public class TestNodeFencer {
         " # the next one will always fail\n" +
         " " + AlwaysFailFencer.class.getName() + "(foo) # <- fails\n" +
         AlwaysSucceedFencer.class.getName() + "(bar) \n");
-    assertTrue(fencer.fence(MOCK_TARGET));
+    assertTrue(fencer.fence(new InetSocketAddress("host", 1234)));
     // One call to each, since top fencer fails
     assertEquals(1, AlwaysFailFencer.fenceCalled);
-    assertSame(MOCK_TARGET, AlwaysFailFencer.fencedSvc);
+    assertEquals("host:1234", AlwaysFailFencer.fencedSvc);
     assertEquals(1, AlwaysSucceedFencer.fenceCalled);
-    assertSame(MOCK_TARGET, AlwaysSucceedFencer.fencedSvc);
+    assertEquals("host:1234", AlwaysSucceedFencer.fencedSvc);
     assertEquals("foo", AlwaysFailFencer.callArgs.get(0));
     assertEquals("bar", AlwaysSucceedFencer.callArgs.get(0));
   }
@@ -91,41 +82,41 @@ public class TestNodeFencer {
   public void testArglessFencer() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer(
         AlwaysSucceedFencer.class.getName());
-    assertTrue(fencer.fence(MOCK_TARGET));
+    assertTrue(fencer.fence(new InetSocketAddress("host", 1234)));
     // One call to each, since top fencer fails
     assertEquals(1, AlwaysSucceedFencer.fenceCalled);
-    assertSame(MOCK_TARGET, AlwaysSucceedFencer.fencedSvc);
+    assertEquals("host:1234", AlwaysSucceedFencer.fencedSvc);
     assertEquals(null, AlwaysSucceedFencer.callArgs.get(0));
   }
 
   @Test
   public void testShortNameShell() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer("shell(true)");
-    assertTrue(fencer.fence(MOCK_TARGET));
+    assertTrue(fencer.fence(new InetSocketAddress("host", 1234)));
   }
 
   @Test
   public void testShortNameSsh() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer("sshfence");
-    assertFalse(fencer.fence(MOCK_TARGET));
+    assertFalse(fencer.fence(new InetSocketAddress("host", 1234)));
   }
 
   @Test
   public void testShortNameSshWithUser() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer("sshfence(user)");
-    assertFalse(fencer.fence(MOCK_TARGET));
+    assertFalse(fencer.fence(new InetSocketAddress("host", 1234)));
   }
 
   @Test
   public void testShortNameSshWithPort() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer("sshfence(:123)");
-    assertFalse(fencer.fence(MOCK_TARGET));
+    assertFalse(fencer.fence(new InetSocketAddress("host", 1234)));
   }
 
   @Test
   public void testShortNameSshWithUserPort() throws BadFencingConfigurationException {
     NodeFencer fencer = setupFencer("sshfence(user:123)");
-    assertFalse(fencer.fence(MOCK_TARGET));
+    assertFalse(fencer.fence(new InetSocketAddress("host", 1234)));
   }
 
   public static NodeFencer setupFencer(String confStr)
@@ -142,12 +133,12 @@ public class TestNodeFencer {
   public static class AlwaysSucceedFencer extends Configured
       implements FenceMethod {
     static int fenceCalled = 0;
-    static HAServiceTarget fencedSvc;
+    static String fencedSvc;
     static List<String> callArgs = Lists.newArrayList();
 
     @Override
-    public boolean tryFence(HAServiceTarget target, String args) {
-      fencedSvc = target;
+    public boolean tryFence(InetSocketAddress serviceAddr, String args) {
+      fencedSvc = serviceAddr.getHostName() + ":" + serviceAddr.getPort();
       callArgs.add(args);
       fenceCalled++;
       return true;
@@ -164,12 +155,12 @@ public class TestNodeFencer {
   public static class AlwaysFailFencer extends Configured
       implements FenceMethod {
     static int fenceCalled = 0;
-    static HAServiceTarget fencedSvc;
+    static String fencedSvc;
     static List<String> callArgs = Lists.newArrayList();
 
     @Override
-    public boolean tryFence(HAServiceTarget target, String args) {
-      fencedSvc = target;
+    public boolean tryFence(InetSocketAddress serviceAddr, String args) {
+      fencedSvc = serviceAddr.getHostName() + ":" + serviceAddr.getPort();
       callArgs.add(args);
       fenceCalled++;
       return false;

+ 12 - 11
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestShellCommandFencer.java

@@ -22,7 +22,6 @@ import static org.junit.Assert.*;
 import java.net.InetSocketAddress;
 
 import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
 import org.apache.hadoop.util.StringUtils;
 import org.junit.Before;
 import org.junit.BeforeClass;
@@ -33,9 +32,6 @@ import static org.mockito.Mockito.spy;
 
 public class TestShellCommandFencer {
   private ShellCommandFencer fencer = createFencer();
-  private static final HAServiceTarget TEST_TARGET =
-      new DummyHAService(HAServiceState.ACTIVE,
-          new InetSocketAddress("host", 1234));
   
   @BeforeClass
   public static void setupLogSpy() {
@@ -61,10 +57,11 @@ public class TestShellCommandFencer {
    */
   @Test
   public void testBasicSuccessFailure() {
-    assertTrue(fencer.tryFence(TEST_TARGET, "echo"));
-    assertFalse(fencer.tryFence(TEST_TARGET, "exit 1"));
+    InetSocketAddress addr = new InetSocketAddress("host", 1234);
+    assertTrue(fencer.tryFence(addr, "echo"));
+    assertFalse(fencer.tryFence(addr, "exit 1"));
     // bad path should also fail
-    assertFalse(fencer.tryFence(TEST_TARGET, "xxxxxxxxxxxx"));
+    assertFalse(fencer.tryFence(addr, "xxxxxxxxxxxx"));
   }
   
   @Test
@@ -101,7 +98,8 @@ public class TestShellCommandFencer {
    */
   @Test
   public void testStdoutLogging() {
-    assertTrue(fencer.tryFence(TEST_TARGET, "echo hello"));
+    InetSocketAddress addr = new InetSocketAddress("host", 1234);
+    assertTrue(fencer.tryFence(addr, "echo hello"));
     Mockito.verify(ShellCommandFencer.LOG).info(
         Mockito.endsWith("echo hello: host:1234 hello"));
   }
@@ -112,7 +110,8 @@ public class TestShellCommandFencer {
    */
   @Test
   public void testStderrLogging() {
-    assertTrue(fencer.tryFence(TEST_TARGET, "echo hello >&2"));
+    InetSocketAddress addr = new InetSocketAddress("host", 1234);
+    assertTrue(fencer.tryFence(addr, "echo hello >&2"));
     Mockito.verify(ShellCommandFencer.LOG).warn(
         Mockito.endsWith("echo hello >&2: host:1234 hello"));
   }
@@ -123,7 +122,8 @@ public class TestShellCommandFencer {
    */
   @Test
   public void testConfAsEnvironment() {
-    fencer.tryFence(TEST_TARGET, "echo $in_fencing_tests");
+    InetSocketAddress addr = new InetSocketAddress("host", 1234);
+    fencer.tryFence(addr, "echo $in_fencing_tests");
     Mockito.verify(ShellCommandFencer.LOG).info(
         Mockito.endsWith("echo $in...ing_tests: host:1234 yessir"));
   }
@@ -136,7 +136,8 @@ public class TestShellCommandFencer {
    */
   @Test(timeout=10000)
   public void testSubprocessInputIsClosed() {
-    assertFalse(fencer.tryFence(TEST_TARGET, "read"));
+    InetSocketAddress addr = new InetSocketAddress("host", 1234);
+    assertFalse(fencer.tryFence(addr, "read"));
   }
   
   @Test

+ 7 - 19
hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/ha/TestSshFenceByTcpPort.java

@@ -23,7 +23,6 @@ import java.net.InetSocketAddress;
 
 import org.apache.commons.logging.impl.Log4JLogger;
 import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
 import org.apache.hadoop.ha.SshFenceByTcpPort.Args;
 import org.apache.log4j.Level;
 import org.junit.Assume;
@@ -35,25 +34,12 @@ public class TestSshFenceByTcpPort {
     ((Log4JLogger)SshFenceByTcpPort.LOG).getLogger().setLevel(Level.ALL);
   }
   
-  private static String TEST_FENCING_HOST = System.getProperty(
+  private String TEST_FENCING_HOST = System.getProperty(
       "test.TestSshFenceByTcpPort.host", "localhost");
-  private static final String TEST_FENCING_PORT = System.getProperty(
+  private String TEST_FENCING_PORT = System.getProperty(
       "test.TestSshFenceByTcpPort.port", "8020");
-  private static final String TEST_KEYFILE = System.getProperty(
+  private final String TEST_KEYFILE = System.getProperty(
       "test.TestSshFenceByTcpPort.key");
-  
-  private static final InetSocketAddress TEST_ADDR =
-    new InetSocketAddress(TEST_FENCING_HOST,
-      Integer.valueOf(TEST_FENCING_PORT));
-  private static final HAServiceTarget TEST_TARGET =
-    new DummyHAService(HAServiceState.ACTIVE, TEST_ADDR);
-  
-  /**
-   *  Connect to Google's DNS server - not running ssh!
-   */
-  private static final HAServiceTarget UNFENCEABLE_TARGET =
-    new DummyHAService(HAServiceState.ACTIVE,
-        new InetSocketAddress("8.8.8.8", 1234));
 
   @Test(timeout=20000)
   public void testFence() throws BadFencingConfigurationException {
@@ -63,7 +49,8 @@ public class TestSshFenceByTcpPort {
     SshFenceByTcpPort fence = new SshFenceByTcpPort();
     fence.setConf(conf);
     assertTrue(fence.tryFence(
-        TEST_TARGET,
+        new InetSocketAddress(TEST_FENCING_HOST,
+                              Integer.valueOf(TEST_FENCING_PORT)),
         null));
   }
 
@@ -78,7 +65,8 @@ public class TestSshFenceByTcpPort {
     conf.setInt(SshFenceByTcpPort.CONF_CONNECT_TIMEOUT_KEY, 3000);
     SshFenceByTcpPort fence = new SshFenceByTcpPort();
     fence.setConf(conf);
-    assertFalse(fence.tryFence(UNFENCEABLE_TARGET, ""));
+    // Connect to Google's DNS server - not running ssh!
+    assertFalse(fence.tryFence(new InetSocketAddress("8.8.8.8", 1234), ""));
   }
   
   @Test

+ 9 - 3
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/tools/DFSHAAdmin.java

@@ -25,8 +25,8 @@ import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.CommonConfigurationKeys;
 import org.apache.hadoop.ha.HAAdmin;
-import org.apache.hadoop.ha.HAServiceTarget;
 import org.apache.hadoop.hdfs.DFSConfigKeys;
+import org.apache.hadoop.hdfs.DFSUtil;
 import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.util.ToolRunner;
 
@@ -65,9 +65,15 @@ public class DFSHAAdmin extends HAAdmin {
    * Try to map the given namenode ID to its service address.
    */
   @Override
-  protected HAServiceTarget resolveTarget(String nnId) {
+  protected String getServiceAddr(String nnId) {
     HdfsConfiguration conf = (HdfsConfiguration)getConf();
-    return new NNHAServiceTarget(conf, nameserviceId, nnId);
+    String serviceAddr = 
+      DFSUtil.getNamenodeServiceAddr(conf, nameserviceId, nnId);
+    if (serviceAddr == null) {
+      throw new IllegalArgumentException(
+          "Unable to determine service address for namenode '" + nnId + "'");
+    }
+    return serviceAddr;
   }
 
   @Override

+ 0 - 84
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/tools/NNHAServiceTarget.java

@@ -1,84 +0,0 @@
-/**
- * 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.hdfs.tools;
-
-import java.net.InetSocketAddress;
-
-import org.apache.hadoop.classification.InterfaceAudience;
-import org.apache.hadoop.ha.BadFencingConfigurationException;
-import org.apache.hadoop.ha.HAServiceTarget;
-import org.apache.hadoop.ha.NodeFencer;
-import org.apache.hadoop.hdfs.DFSUtil;
-import org.apache.hadoop.hdfs.HdfsConfiguration;
-import org.apache.hadoop.hdfs.server.namenode.NameNode;
-import org.apache.hadoop.net.NetUtils;
-
-/**
- * One of the NN NameNodes acting as the target of an administrative command
- * (e.g. failover).
- */
-@InterfaceAudience.Private
-public class NNHAServiceTarget extends HAServiceTarget {
-
-  private final InetSocketAddress addr;
-  private NodeFencer fencer;
-  private BadFencingConfigurationException fenceConfigError;
-
-  public NNHAServiceTarget(HdfsConfiguration conf,
-      String nsId, String nnId) {
-    String serviceAddr = 
-      DFSUtil.getNamenodeServiceAddr(conf, nsId, nnId);
-    if (serviceAddr == null) {
-      throw new IllegalArgumentException(
-          "Unable to determine service address for namenode '" + nnId + "'");
-    }
-    this.addr = NetUtils.createSocketAddr(serviceAddr,
-        NameNode.DEFAULT_PORT);
-    try {
-      this.fencer = NodeFencer.create(conf);
-    } catch (BadFencingConfigurationException e) {
-      this.fenceConfigError = e;
-    }
-  }
-
-  /**
-   * @return the NN's IPC address.
-   */
-  @Override
-  public InetSocketAddress getAddress() {
-    return addr;
-  }
-
-  @Override
-  public void checkFencingConfigured() throws BadFencingConfigurationException {
-    if (fenceConfigError != null) {
-      throw fenceConfigError;
-    }
-  }
-  
-  @Override
-  public NodeFencer getFencer() {
-    return fencer;
-  }
-  
-  @Override
-  public String toString() {
-    return "NameNode at " + addr;
-  }
-
-}

+ 3 - 12
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/tools/TestDFSHAAdmin.java

@@ -32,7 +32,6 @@ import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.ha.HAServiceProtocol;
 import org.apache.hadoop.ha.HAServiceProtocol.HAServiceState;
 import org.apache.hadoop.ha.HAServiceStatus;
-import org.apache.hadoop.ha.HAServiceTarget;
 import org.apache.hadoop.ha.HealthCheckFailedException;
 import org.apache.hadoop.ha.NodeFencer;
 
@@ -80,18 +79,10 @@ public class TestDFSHAAdmin {
   public void setup() throws IOException {
     mockProtocol = Mockito.mock(HAServiceProtocol.class);
     tool = new DFSHAAdmin() {
-
       @Override
-      protected HAServiceTarget resolveTarget(String nnId) {
-        HAServiceTarget target = super.resolveTarget(nnId);
-        HAServiceTarget spy = Mockito.spy(target);
-        // OVerride the target to return our mock protocol
-        try {
-          Mockito.doReturn(mockProtocol).when(spy).getProxy();
-        } catch (IOException e) {
-          throw new AssertionError(e); // mock setup doesn't really throw
-        }
-        return spy;
+      protected HAServiceProtocol getProtocol(String serviceId) throws IOException {
+        getServiceAddr(serviceId);
+        return mockProtocol;
       }
     };
     tool.setConf(getHAConf());