|
@@ -137,6 +137,7 @@ import org.apache.hadoop.tools.protocolPB.GetUserMappingsProtocolServerSideTrans
|
|
|
import org.apache.hadoop.util.VersionInfo;
|
|
|
import org.apache.hadoop.util.VersionUtil;
|
|
|
|
|
|
+import com.google.common.annotations.VisibleForTesting;
|
|
|
import com.google.protobuf.BlockingService;
|
|
|
|
|
|
/**
|
|
@@ -219,6 +220,13 @@ class NameNodeRpcServer implements NamenodeProtocols {
|
|
|
|
|
|
InetSocketAddress serviceRpcAddr = nn.getServiceRpcServerAddress(conf);
|
|
|
if (serviceRpcAddr != null) {
|
|
|
+ String bindHost = nn.getServiceRpcServerBindHost(conf);
|
|
|
+ if (bindHost == null) {
|
|
|
+ bindHost = serviceRpcAddr.getHostName();
|
|
|
+ }
|
|
|
+ LOG.info("Service RPC server is binding to " + bindHost + ":" +
|
|
|
+ serviceRpcAddr.getPort());
|
|
|
+
|
|
|
int serviceHandlerCount =
|
|
|
conf.getInt(DFS_NAMENODE_SERVICE_HANDLER_COUNT_KEY,
|
|
|
DFS_NAMENODE_SERVICE_HANDLER_COUNT_DEFAULT);
|
|
@@ -226,7 +234,7 @@ class NameNodeRpcServer implements NamenodeProtocols {
|
|
|
.setProtocol(
|
|
|
org.apache.hadoop.hdfs.protocolPB.ClientNamenodeProtocolPB.class)
|
|
|
.setInstance(clientNNPbService)
|
|
|
- .setBindAddress(serviceRpcAddr.getHostName())
|
|
|
+ .setBindAddress(bindHost)
|
|
|
.setPort(serviceRpcAddr.getPort()).setNumHandlers(serviceHandlerCount)
|
|
|
.setVerbose(false)
|
|
|
.setSecretManager(namesystem.getDelegationTokenSecretManager())
|
|
@@ -246,17 +254,26 @@ class NameNodeRpcServer implements NamenodeProtocols {
|
|
|
DFSUtil.addPBProtocol(conf, GetUserMappingsProtocolPB.class,
|
|
|
getUserMappingService, serviceRpcServer);
|
|
|
|
|
|
- serviceRPCAddress = serviceRpcServer.getListenerAddress();
|
|
|
+ // Update the address with the correct port
|
|
|
+ InetSocketAddress listenAddr = serviceRpcServer.getListenerAddress();
|
|
|
+ serviceRPCAddress = new InetSocketAddress(
|
|
|
+ serviceRpcAddr.getHostName(), listenAddr.getPort());
|
|
|
nn.setRpcServiceServerAddress(conf, serviceRPCAddress);
|
|
|
} else {
|
|
|
serviceRpcServer = null;
|
|
|
serviceRPCAddress = null;
|
|
|
}
|
|
|
InetSocketAddress rpcAddr = nn.getRpcServerAddress(conf);
|
|
|
+ String bindHost = nn.getRpcServerBindHost(conf);
|
|
|
+ if (bindHost == null) {
|
|
|
+ bindHost = rpcAddr.getHostName();
|
|
|
+ }
|
|
|
+ LOG.info("RPC server is binding to " + bindHost + ":" + rpcAddr.getPort());
|
|
|
+
|
|
|
this.clientRpcServer = new RPC.Builder(conf)
|
|
|
.setProtocol(
|
|
|
org.apache.hadoop.hdfs.protocolPB.ClientNamenodeProtocolPB.class)
|
|
|
- .setInstance(clientNNPbService).setBindAddress(rpcAddr.getHostName())
|
|
|
+ .setInstance(clientNNPbService).setBindAddress(bindHost)
|
|
|
.setPort(rpcAddr.getPort()).setNumHandlers(handlerCount)
|
|
|
.setVerbose(false)
|
|
|
.setSecretManager(namesystem.getDelegationTokenSecretManager()).build();
|
|
@@ -286,7 +303,9 @@ class NameNodeRpcServer implements NamenodeProtocols {
|
|
|
}
|
|
|
|
|
|
// The rpc-server port can be ephemeral... ensure we have the correct info
|
|
|
- clientRpcAddress = clientRpcServer.getListenerAddress();
|
|
|
+ InetSocketAddress listenAddr = clientRpcServer.getListenerAddress();
|
|
|
+ clientRpcAddress = new InetSocketAddress(
|
|
|
+ rpcAddr.getHostName(), listenAddr.getPort());
|
|
|
nn.setRpcServerAddress(conf, clientRpcAddress);
|
|
|
|
|
|
minimumDataNodeVersion = conf.get(
|
|
@@ -310,6 +329,12 @@ class NameNodeRpcServer implements NamenodeProtocols {
|
|
|
NSQuotaExceededException.class,
|
|
|
DSQuotaExceededException.class);
|
|
|
}
|
|
|
+
|
|
|
+ /** Allow access to the client RPC server for testing */
|
|
|
+ @VisibleForTesting
|
|
|
+ RPC.Server getClientRpcServer() {
|
|
|
+ return clientRpcServer;
|
|
|
+ }
|
|
|
|
|
|
/**
|
|
|
* Start client and service RPC servers.
|