001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.zookeeper;
019
020import java.util.ArrayList;
021import java.util.Collections;
022import java.util.List;
023import org.apache.hadoop.conf.Configuration;
024import org.apache.hadoop.hbase.HConstants;
025import org.apache.hadoop.hbase.NotAllMetaRegionsOnlineException;
026import org.apache.hadoop.hbase.ServerName;
027import org.apache.hadoop.hbase.client.RegionInfo;
028import org.apache.hadoop.hbase.client.RegionInfoBuilder;
029import org.apache.hadoop.hbase.client.RegionReplicaUtil;
030import org.apache.hadoop.hbase.exceptions.DeserializationException;
031import org.apache.hadoop.hbase.master.RegionState;
032import org.apache.hadoop.hbase.util.Pair;
033import org.apache.yetus.audience.InterfaceAudience;
034import org.apache.zookeeper.KeeperException;
035import org.slf4j.Logger;
036import org.slf4j.LoggerFactory;
037import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
038import org.apache.hadoop.hbase.shaded.protobuf.generated.ZooKeeperProtos.MetaRegionServer;
039
040/**
041 * Utility class to perform operation (get/wait for/verify/set/delete) on znode in ZooKeeper which
042 * keeps hbase:meta region server location.
043 * <p/>
044 * Stateless class with a bunch of static methods. Doesn't manage resources passed in (e.g.
045 * Connection, ZKWatcher etc).
046 * <p/>
047 * Meta region location is set by <code>RegionServerServices</code>. This class doesn't use ZK
048 * watchers, rather accesses ZK directly.
049 * <p/>
050 * TODO: rewrite using RPC calls to master to find out about hbase:meta.
051 */
052@InterfaceAudience.Private
053public final class MetaTableLocator {
054  private static final Logger LOG = LoggerFactory.getLogger(MetaTableLocator.class);
055
056  private MetaTableLocator() {
057  }
058
059  /**
060   * @param zkw ZooKeeper watcher to be used
061   * @return meta table regions and their locations.
062   */
063  public static List<Pair<RegionInfo, ServerName>> getMetaRegionsAndLocations(ZKWatcher zkw) {
064    return getMetaRegionsAndLocations(zkw, RegionInfo.DEFAULT_REPLICA_ID);
065  }
066
067  /**
068   * Gets the meta regions and their locations for the given path and replica ID.
069   *
070   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
071   * @param replicaId the ID of the replica
072   * @return meta table regions and their locations.
073   */
074  public static List<Pair<RegionInfo, ServerName>> getMetaRegionsAndLocations(ZKWatcher zkw,
075      int replicaId) {
076    ServerName serverName = getMetaRegionLocation(zkw, replicaId);
077    List<Pair<RegionInfo, ServerName>> list = new ArrayList<>(1);
078    list.add(new Pair<>(RegionReplicaUtil.getRegionInfoForReplica(
079        RegionInfoBuilder.FIRST_META_REGIONINFO, replicaId), serverName));
080    return list;
081  }
082
083  /**
084   * Gets the meta regions for the given path with the default replica ID.
085   *
086   * @param zkw ZooKeeper watcher to be used
087   * @return List of meta regions
088   */
089  public static List<RegionInfo> getMetaRegions(ZKWatcher zkw) {
090    return getMetaRegions(zkw, RegionInfo.DEFAULT_REPLICA_ID);
091  }
092
093  /**
094   * Gets the meta regions for the given path and replica ID.
095   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
096   * @param replicaId the ID of the replica
097   * @return List of meta regions
098   */
099  public static List<RegionInfo> getMetaRegions(ZKWatcher zkw, int replicaId) {
100    List<Pair<RegionInfo, ServerName>> result;
101    result = getMetaRegionsAndLocations(zkw, replicaId);
102    return getListOfRegionInfos(result);
103  }
104
105  private static List<RegionInfo> getListOfRegionInfos(
106      final List<Pair<RegionInfo, ServerName>> pairs) {
107    if (pairs == null || pairs.isEmpty()) {
108      return Collections.emptyList();
109    }
110
111    List<RegionInfo> result = new ArrayList<>(pairs.size());
112    for (Pair<RegionInfo, ServerName> pair : pairs) {
113      result.add(pair.getFirst());
114    }
115    return result;
116  }
117
118  /**
119   * Gets the meta region location, if available.  Does not block.
120   * @param zkw zookeeper connection to use
121   * @return server name or null if we failed to get the data.
122   */
123  public static ServerName getMetaRegionLocation(final ZKWatcher zkw) {
124    try {
125      RegionState state = getMetaRegionState(zkw);
126      return state.isOpened() ? state.getServerName() : null;
127    } catch (KeeperException ke) {
128      return null;
129    }
130  }
131
132  /**
133   * Gets the meta region location, if available.  Does not block.
134   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
135   * @param replicaId the ID of the replica
136   * @return server name
137   */
138  public static ServerName getMetaRegionLocation(final ZKWatcher zkw, int replicaId) {
139    try {
140      RegionState state = getMetaRegionState(zkw, replicaId);
141      return state.isOpened() ? state.getServerName() : null;
142    } catch (KeeperException ke) {
143      return null;
144    }
145  }
146
147  /**
148   * Gets the meta region location, if available, and waits for up to the specified timeout if not
149   * immediately available. Given the zookeeper notification could be delayed, we will try to get
150   * the latest data.
151   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
152   * @param timeout maximum time to wait, in millis
153   * @return server name for server hosting meta region formatted as per {@link ServerName}, or null
154   *         if none available
155   * @throws InterruptedException if interrupted while waiting
156   * @throws NotAllMetaRegionsOnlineException if a meta or root region is not online
157   */
158  public static ServerName waitMetaRegionLocation(ZKWatcher zkw, long timeout)
159      throws InterruptedException, NotAllMetaRegionsOnlineException {
160    return waitMetaRegionLocation(zkw, RegionInfo.DEFAULT_REPLICA_ID, timeout);
161  }
162
163  /**
164   * Gets the meta region location, if available, and waits for up to the specified timeout if not
165   * immediately available. Given the zookeeper notification could be delayed, we will try to get
166   * the latest data.
167   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
168   * @param replicaId the ID of the replica
169   * @param timeout maximum time to wait, in millis
170   * @return server name for server hosting meta region formatted as per {@link ServerName}, or null
171   *         if none available
172   * @throws InterruptedException if waiting for the socket operation fails
173   * @throws NotAllMetaRegionsOnlineException if a meta or root region is not online
174   */
175  public static ServerName waitMetaRegionLocation(ZKWatcher zkw, int replicaId, long timeout)
176      throws InterruptedException, NotAllMetaRegionsOnlineException {
177    try {
178      if (ZKUtil.checkExists(zkw, zkw.getZNodePaths().baseZNode) == -1) {
179        String errorMsg = "Check the value configured in 'zookeeper.znode.parent'. " +
180          "There could be a mismatch with the one configured in the master.";
181        LOG.error(errorMsg);
182        throw new IllegalArgumentException(errorMsg);
183      }
184    } catch (KeeperException e) {
185      throw new IllegalStateException("KeeperException while trying to check baseZNode:", e);
186    }
187    ServerName sn = blockUntilAvailable(zkw, replicaId, timeout);
188
189    if (sn == null) {
190      throw new NotAllMetaRegionsOnlineException("Timed out; " + timeout + "ms");
191    }
192
193    return sn;
194  }
195
196  /**
197   * Sets the location of <code>hbase:meta</code> in ZooKeeper to the
198   * specified server address.
199   * @param zookeeper zookeeper reference
200   * @param serverName The server hosting <code>hbase:meta</code>
201   * @param state The region transition state
202   * @throws KeeperException unexpected zookeeper exception
203   */
204  public static void setMetaLocation(ZKWatcher zookeeper,
205      ServerName serverName, RegionState.State state) throws KeeperException {
206    setMetaLocation(zookeeper, serverName, RegionInfo.DEFAULT_REPLICA_ID, state);
207  }
208
209  /**
210   * Sets the location of <code>hbase:meta</code> in ZooKeeper to the specified server address.
211   * @param zookeeper reference to the {@link ZKWatcher} which also contains configuration and
212   *                  operation
213   * @param serverName the name of the server
214   * @param replicaId the ID of the replica
215   * @param state the state of the region
216   * @throws KeeperException if a ZooKeeper operation fails
217   */
218  public static void setMetaLocation(ZKWatcher zookeeper, ServerName serverName, int replicaId,
219      RegionState.State state) throws KeeperException {
220    if (serverName == null) {
221      LOG.warn("Tried to set null ServerName in hbase:meta; skipping -- ServerName required");
222      return;
223    }
224    LOG.info("Setting hbase:meta (replicaId={}) location in ZooKeeper as {}", replicaId,
225      serverName);
226    // Make the MetaRegionServer pb and then get its bytes and save this as
227    // the znode content.
228    MetaRegionServer pbrsr = MetaRegionServer.newBuilder()
229      .setServer(ProtobufUtil.toServerName(serverName))
230      .setRpcVersion(HConstants.RPC_CURRENT_VERSION)
231      .setState(state.convert()).build();
232    byte[] data = ProtobufUtil.prependPBMagic(pbrsr.toByteArray());
233    try {
234      ZKUtil.setData(zookeeper,
235          zookeeper.getZNodePaths().getZNodeForReplica(replicaId), data);
236    } catch(KeeperException.NoNodeException nne) {
237      if (replicaId == RegionInfo.DEFAULT_REPLICA_ID) {
238        LOG.debug("META region location doesn't exist, create it");
239      } else {
240        LOG.debug("META region location doesn't exist for replicaId=" + replicaId +
241            ", create it");
242      }
243      ZKUtil.createAndWatch(zookeeper, zookeeper.getZNodePaths().getZNodeForReplica(replicaId),
244              data);
245    }
246  }
247
248  /**
249   * Load the meta region state from the meta server ZNode.
250   */
251  public static RegionState getMetaRegionState(ZKWatcher zkw) throws KeeperException {
252    return getMetaRegionState(zkw, RegionInfo.DEFAULT_REPLICA_ID);
253  }
254
255  /**
256   * Load the meta region state from the meta region server ZNode.
257   *
258   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
259   * @param replicaId the ID of the replica
260   * @return regionstate
261   * @throws KeeperException if a ZooKeeper operation fails
262   */
263  public static RegionState getMetaRegionState(ZKWatcher zkw, int replicaId)
264      throws KeeperException {
265    RegionState regionState = null;
266    try {
267      byte[] data = ZKUtil.getData(zkw, zkw.getZNodePaths().getZNodeForReplica(replicaId));
268      regionState = ProtobufUtil.parseMetaRegionStateFrom(data, replicaId);
269    } catch (DeserializationException e) {
270      throw ZKUtil.convert(e);
271    } catch (InterruptedException e) {
272      Thread.currentThread().interrupt();
273    }
274    return regionState;
275  }
276
277  /**
278   * Deletes the location of <code>hbase:meta</code> in ZooKeeper.
279   * @param zookeeper zookeeper reference
280   * @throws KeeperException unexpected zookeeper exception
281   */
282  public static void deleteMetaLocation(ZKWatcher zookeeper)
283    throws KeeperException {
284    deleteMetaLocation(zookeeper, RegionInfo.DEFAULT_REPLICA_ID);
285  }
286
287  public static void deleteMetaLocation(ZKWatcher zookeeper, int replicaId)
288    throws KeeperException {
289    if (replicaId == RegionInfo.DEFAULT_REPLICA_ID) {
290      LOG.info("Deleting hbase:meta region location in ZooKeeper");
291    } else {
292      LOG.info("Deleting hbase:meta for {} region location in ZooKeeper", replicaId);
293    }
294    try {
295      // Just delete the node.  Don't need any watches.
296      ZKUtil.deleteNode(zookeeper, zookeeper.getZNodePaths().getZNodeForReplica(replicaId));
297    } catch(KeeperException.NoNodeException nne) {
298      // Has already been deleted
299    }
300  }
301  /**
302   * Wait until the primary meta region is available. Get the secondary locations as well but don't
303   * block for those.
304   *
305   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and operation
306   * @param timeout maximum time to wait in millis
307   * @param conf the {@link Configuration} to use
308   * @return ServerName or null if we timed out.
309   * @throws InterruptedException if waiting for the socket operation fails
310   */
311  public static List<ServerName> blockUntilAvailable(final ZKWatcher zkw, final long timeout,
312      Configuration conf) throws InterruptedException {
313    int numReplicasConfigured = 1;
314
315    List<ServerName> servers = new ArrayList<>();
316    // Make the blocking call first so that we do the wait to know
317    // the znodes are all in place or timeout.
318    ServerName server = blockUntilAvailable(zkw, timeout);
319
320    if (server == null) {
321      return null;
322    }
323
324    servers.add(server);
325
326    try {
327      List<String> metaReplicaNodes = zkw.getMetaReplicaNodes();
328      numReplicasConfigured = metaReplicaNodes.size();
329    } catch (KeeperException e) {
330      LOG.warn("Got ZK exception {}", e);
331    }
332    for (int replicaId = 1; replicaId < numReplicasConfigured; replicaId++) {
333      // return all replica locations for the meta
334      servers.add(getMetaRegionLocation(zkw, replicaId));
335    }
336    return servers;
337  }
338
339  /**
340   * Wait until the meta region is available and is not in transition.
341   * @param zkw zookeeper connection to use
342   * @param timeout maximum time to wait, in millis
343   * @return ServerName or null if we timed out.
344   * @throws InterruptedException if waiting for the socket operation fails
345   */
346  public static ServerName blockUntilAvailable(final ZKWatcher zkw, final long timeout)
347      throws InterruptedException {
348    return blockUntilAvailable(zkw, RegionInfo.DEFAULT_REPLICA_ID, timeout);
349  }
350
351  /**
352   * Wait until the meta region is available and is not in transition.
353   * @param zkw reference to the {@link ZKWatcher} which also contains configuration and constants
354   * @param replicaId the ID of the replica
355   * @param timeout maximum time to wait in millis
356   * @return ServerName or null if we timed out.
357   * @throws InterruptedException if waiting for the socket operation fails
358   */
359  public static ServerName blockUntilAvailable(final ZKWatcher zkw, int replicaId,
360      final long timeout) throws InterruptedException {
361    if (timeout < 0) {
362      throw new IllegalArgumentException();
363    }
364
365    if (zkw == null) {
366      throw new IllegalArgumentException();
367    }
368
369    long startTime = System.currentTimeMillis();
370    ServerName sn = null;
371    while (true) {
372      sn = getMetaRegionLocation(zkw, replicaId);
373      if (sn != null ||
374        (System.currentTimeMillis() - startTime) > timeout - HConstants.SOCKET_RETRY_WAIT_MS) {
375        break;
376      }
377      Thread.sleep(HConstants.SOCKET_RETRY_WAIT_MS);
378    }
379    return sn;
380  }
381}