View Javadoc

1   /**
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  package org.apache.hadoop.hbase.zookeeper;
20  
21  import java.io.IOException;
22  import java.util.ArrayList;
23  import java.util.List;
24  import java.util.NavigableMap;
25  import java.util.NavigableSet;
26  import java.util.TreeMap;
27  import java.util.TreeSet;
28  
29  import org.apache.commons.logging.Log;
30  import org.apache.commons.logging.LogFactory;
31  import org.apache.hadoop.classification.InterfaceAudience;
32  import org.apache.hadoop.hbase.Abortable;
33  import org.apache.hadoop.hbase.ServerName;
34  import org.apache.hadoop.hbase.master.ServerManager;
35  import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
36  import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionServerInfo;
37  import org.apache.hadoop.hbase.util.Bytes;
38  import org.apache.zookeeper.KeeperException;
39  
40  /**
41   * Tracks the online region servers via ZK.
42   *
43   * <p>Handling of new RSs checking in is done via RPC.  This class
44   * is only responsible for watching for expired nodes.  It handles
45   * listening for changes in the RS node list and watching each node.
46   *
47   * <p>If an RS node gets deleted, this automatically handles calling of
48   * {@link ServerManager#expireServer(ServerName)}
49   */
50  @InterfaceAudience.Private
51  public class RegionServerTracker extends ZooKeeperListener {
52    private static final Log LOG = LogFactory.getLog(RegionServerTracker.class);
53    private NavigableMap<ServerName, RegionServerInfo> regionServers = 
54  		  new TreeMap<ServerName, RegionServerInfo>();
55    private ServerManager serverManager;
56    private Abortable abortable;
57  
58    public RegionServerTracker(ZooKeeperWatcher watcher,
59        Abortable abortable, ServerManager serverManager) {
60      super(watcher);
61      this.abortable = abortable;
62      this.serverManager = serverManager;
63    }
64  
65    /**
66     * Starts the tracking of online RegionServers.
67     *
68     * <p>All RSs will be tracked after this method is called.
69     *
70     * @throws KeeperException
71     * @throws IOException
72     */
73    public void start() throws KeeperException, IOException {
74      watcher.registerListener(this);
75      List<String> servers =
76        ZKUtil.listChildrenAndWatchThem(watcher, watcher.rsZNode);
77      add(servers);
78    }
79  
80    private void add(final List<String> servers) throws IOException {
81      synchronized(this.regionServers) {
82        this.regionServers.clear();
83        for (String n: servers) {
84          ServerName sn = ServerName.parseServerName(ZKUtil.getNodeName(n));
85          if (regionServers.get(sn) == null) {
86            RegionServerInfo.Builder rsInfoBuilder = RegionServerInfo.newBuilder();
87            try {
88              String nodePath = ZKUtil.joinZNode(watcher.rsZNode, n);
89              byte[] data = ZKUtil.getData(watcher, nodePath);
90              if (LOG.isDebugEnabled()) {
91                LOG.debug("RS node: " + nodePath + " data: " + Bytes.toString(data));
92              }
93              if (data != null && data.length > 0 && ProtobufUtil.isPBMagicPrefix(data)) {
94                int magicLen = ProtobufUtil.lengthOfPBMagic();
95                rsInfoBuilder.mergeFrom(data, magicLen, data.length - magicLen);
96              }
97            } catch (KeeperException e) {
98              LOG.warn("Get Rs info port from ephemeral node", e);
99            } catch (IOException e) {
100             LOG.warn("Illegal data from ephemeral node", e);
101           }
102           this.regionServers.put(sn, rsInfoBuilder.build());
103         }
104       }
105     }
106   }
107 
108   private void remove(final ServerName sn) {
109     synchronized(this.regionServers) {
110       this.regionServers.remove(sn);
111     }
112   }
113 
114   @Override
115   public void nodeDeleted(String path) {
116     if (path.startsWith(watcher.rsZNode)) {
117       String serverName = ZKUtil.getNodeName(path);
118       LOG.info("RegionServer ephemeral node deleted, processing expiration [" +
119         serverName + "]");
120       ServerName sn = ServerName.parseServerName(serverName);
121       if (!serverManager.isServerOnline(sn)) {
122         LOG.warn(serverName.toString() + " is not online or isn't known to the master."+
123          "The latter could be caused by a DNS misconfiguration.");
124         return;
125       }
126       remove(sn);
127       this.serverManager.expireServer(sn);
128     }
129   }
130 
131   @Override
132   public void nodeChildrenChanged(String path) {
133     if (path.equals(watcher.rsZNode)) {
134       try {
135         List<String> servers =
136           ZKUtil.listChildrenAndWatchThem(watcher, watcher.rsZNode);
137         add(servers);
138       } catch (IOException e) {
139         abortable.abort("Unexpected zk exception getting RS nodes", e);
140       } catch (KeeperException e) {
141         abortable.abort("Unexpected zk exception getting RS nodes", e);
142       }
143     }
144   }
145 
146   public RegionServerInfo getRegionServerInfo(final ServerName sn) {
147     return regionServers.get(sn);
148   }
149   
150   /**
151    * Gets the online servers.
152    * @return list of online servers
153    */
154   public List<ServerName> getOnlineServers() {
155     synchronized (this.regionServers) {
156       return new ArrayList<ServerName>(this.regionServers.keySet());
157     }
158   }
159 }