HBASE-26474 Implement connection-level attributes (addendum)
[hbase.git] / hbase-client / src / main / java / org / apache / hadoop / hbase / client / MasterRegistry.java
blob05773d0b4195424cd227be7e58d776b4a41cd311
1 /*
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.client;
20 import static org.apache.hadoop.hbase.HConstants.MASTER_ADDRS_KEY;
21 import static org.apache.hadoop.hbase.util.DNS.getHostname;
23 import com.google.errorprone.annotations.RestrictedApi;
24 import java.io.IOException;
25 import java.net.UnknownHostException;
26 import java.util.HashSet;
27 import java.util.Set;
28 import java.util.concurrent.CompletableFuture;
29 import java.util.stream.Collectors;
30 import org.apache.hadoop.conf.Configuration;
31 import org.apache.hadoop.hbase.HConstants;
32 import org.apache.hadoop.hbase.ServerName;
33 import org.apache.hadoop.hbase.util.DNS.ServerType;
34 import org.apache.yetus.audience.InterfaceAudience;
36 import org.apache.hbase.thirdparty.com.google.common.base.Preconditions;
37 import org.apache.hbase.thirdparty.com.google.common.base.Strings;
38 import org.apache.hbase.thirdparty.com.google.common.net.HostAndPort;
40 import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
41 import org.apache.hadoop.hbase.shaded.protobuf.generated.RegistryProtos.GetMastersRequest;
42 import org.apache.hadoop.hbase.shaded.protobuf.generated.RegistryProtos.GetMastersResponse;
44 /**
45 * Master based registry implementation. Makes RPCs to the configured master addresses from config
46 * {@value org.apache.hadoop.hbase.HConstants#MASTER_ADDRS_KEY}.
47 * <p/>
48 * It supports hedged reads, set the fan out of the requests batch by
49 * {@link #MASTER_REGISTRY_HEDGED_REQS_FANOUT_KEY} to a value greater than {@code 1} will enable
50 * it(the default value is {@link AbstractRpcBasedConnectionRegistry#HEDGED_REQS_FANOUT_DEFAULT}).
51 * <p/>
52 * @deprecated Since 2.5.0, will be removed in 4.0.0. Use {@link RpcConnectionRegistry} instead.
54 @Deprecated
55 @InterfaceAudience.Private
56 public class MasterRegistry extends AbstractRpcBasedConnectionRegistry {
58 /** Configuration key that controls the fan out of requests **/
59 public static final String MASTER_REGISTRY_HEDGED_REQS_FANOUT_KEY =
60 "hbase.client.master_registry.hedged.fanout";
62 public static final String MASTER_REGISTRY_INITIAL_REFRESH_DELAY_SECS =
63 "hbase.client.master_registry.initial_refresh_delay_secs";
65 public static final String MASTER_REGISTRY_PERIODIC_REFRESH_INTERVAL_SECS =
66 "hbase.client.master_registry.refresh_interval_secs";
68 public static final String MASTER_REGISTRY_MIN_SECS_BETWEEN_REFRESHES =
69 "hbase.client.master_registry.min_secs_between_refreshes";
71 private static final String MASTER_ADDRS_CONF_SEPARATOR = ",";
73 /**
74 * Parses the list of master addresses from the provided configuration. Supported format is comma
75 * separated host[:port] values. If no port number if specified, default master port is assumed.
76 * @param conf Configuration to parse from.
78 public static Set<ServerName> parseMasterAddrs(Configuration conf) throws UnknownHostException {
79 Set<ServerName> masterAddrs = new HashSet<>();
80 String configuredMasters = getMasterAddr(conf);
81 for (String masterAddr : configuredMasters.split(MASTER_ADDRS_CONF_SEPARATOR)) {
82 HostAndPort masterHostPort =
83 HostAndPort.fromString(masterAddr.trim()).withDefaultPort(HConstants.DEFAULT_MASTER_PORT);
84 masterAddrs.add(ServerName.valueOf(masterHostPort.toString(), ServerName.NON_STARTCODE));
86 Preconditions.checkArgument(!masterAddrs.isEmpty(), "At least one master address is needed");
87 return masterAddrs;
90 private final String connectionString;
92 MasterRegistry(Configuration conf) throws IOException {
93 super(conf, MASTER_REGISTRY_HEDGED_REQS_FANOUT_KEY, MASTER_REGISTRY_INITIAL_REFRESH_DELAY_SECS,
94 MASTER_REGISTRY_PERIODIC_REFRESH_INTERVAL_SECS, MASTER_REGISTRY_MIN_SECS_BETWEEN_REFRESHES);
95 connectionString = getConnectionString(conf);
98 @Override
99 protected Set<ServerName> getBootstrapNodes(Configuration conf) throws IOException {
100 return parseMasterAddrs(conf);
103 @Override
104 protected CompletableFuture<Set<ServerName>> fetchEndpoints() {
105 return getMasters();
108 @Override
109 public String getConnectionString() {
110 return connectionString;
113 static String getConnectionString(Configuration conf) throws UnknownHostException {
114 return getMasterAddr(conf);
118 * Builds the default master address end point if it is not specified in the configuration.
119 * <p/>
120 * Will be called in {@code HBaseTestingUtility}.
122 public static String getMasterAddr(Configuration conf) throws UnknownHostException {
123 String masterAddrFromConf = conf.get(MASTER_ADDRS_KEY);
124 if (!Strings.isNullOrEmpty(masterAddrFromConf)) {
125 return masterAddrFromConf;
127 String hostname = getHostname(conf, ServerType.MASTER);
128 int port = conf.getInt(HConstants.MASTER_PORT, HConstants.DEFAULT_MASTER_PORT);
129 return String.format("%s:%d", hostname, port);
132 private static Set<ServerName> transformServerNames(GetMastersResponse resp) {
133 return resp.getMasterServersList().stream()
134 .map(s -> ProtobufUtil.toServerName(s.getServerName())).collect(Collectors.toSet());
137 @RestrictedApi(explanation = "Should only be called in tests", link = "",
138 allowedOnPath = ".*/(.*/MasterRegistry.java|src/test/.*)")
139 CompletableFuture<Set<ServerName>> getMasters() {
140 return this
141 .<GetMastersResponse> call(
142 (c, s, d) -> s.getMasters(c, GetMastersRequest.getDefaultInstance(), d),
143 r -> r.getMasterServersCount() != 0, "getMasters()")
144 .thenApply(MasterRegistry::transformServerNames);