1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19 package org.apache.hadoop.hbase.zookeeper;
20
21 import java.io.IOException;
22 import java.io.InterruptedIOException;
23 import java.util.ArrayList;
24 import java.util.List;
25 import java.util.NavigableMap;
26 import java.util.TreeMap;
27
28 import org.apache.commons.logging.Log;
29 import org.apache.commons.logging.LogFactory;
30 import org.apache.hadoop.hbase.Server;
31 import org.apache.hadoop.hbase.ServerName;
32 import org.apache.hadoop.hbase.classification.InterfaceAudience;
33 import org.apache.hadoop.hbase.master.ServerManager;
34 import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
35 import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos.RegionServerInfo;
36 import org.apache.zookeeper.KeeperException;
37
38
39
40
41
42
43
44
45
46
47
48 @InterfaceAudience.Private
49 public class RegionServerTracker extends ZooKeeperListener {
50 private static final Log LOG = LogFactory.getLog(RegionServerTracker.class);
51 private NavigableMap<ServerName, RegionServerInfo> regionServers =
52 new TreeMap<ServerName, RegionServerInfo>();
53 private ServerManager serverManager;
54 private Server server;
55
56 public RegionServerTracker(ZooKeeperWatcher watcher,
57 Server server, ServerManager serverManager) {
58 super(watcher);
59 this.server = server;
60 this.serverManager = serverManager;
61 }
62
63
64
65
66
67
68
69
70
71 public void start() throws KeeperException, IOException {
72 watcher.registerListener(this);
73 List<String> servers =
74 ZKUtil.listChildrenAndWatchThem(watcher, watcher.rsZNode);
75 add(servers);
76 }
77
78 private void add(final List<String> servers) throws IOException {
79 synchronized(this.regionServers) {
80 this.regionServers.clear();
81 for (String n: servers) {
82 ServerName sn = ServerName.parseServerName(ZKUtil.getNodeName(n));
83 if (regionServers.get(sn) == null) {
84 RegionServerInfo.Builder rsInfoBuilder = RegionServerInfo.newBuilder();
85 try {
86 String nodePath = ZKUtil.joinZNode(watcher.rsZNode, n);
87 byte[] data = ZKUtil.getData(watcher, nodePath);
88 if (data != null && data.length > 0 && ProtobufUtil.isPBMagicPrefix(data)) {
89 int magicLen = ProtobufUtil.lengthOfPBMagic();
90 ProtobufUtil.mergeFrom(rsInfoBuilder, data, magicLen, data.length - magicLen);
91 }
92 if (LOG.isDebugEnabled()) {
93 LOG.debug("Added tracking of RS " + nodePath);
94 }
95 } catch (KeeperException e) {
96 LOG.warn("Get Rs info port from ephemeral node", e);
97 } catch (IOException e) {
98 LOG.warn("Illegal data from ephemeral node", e);
99 } catch (InterruptedException e) {
100 throw new InterruptedIOException();
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 && !server.isAborted() && !server.isStopped()) {
135 try {
136 List<String> servers =
137 ZKUtil.listChildrenAndWatchThem(watcher, watcher.rsZNode);
138 add(servers);
139 } catch (IOException e) {
140 server.abort("Unexpected zk exception getting RS nodes", e);
141 } catch (KeeperException e) {
142 server.abort("Unexpected zk exception getting RS nodes", e);
143 }
144 }
145 }
146
147 public RegionServerInfo getRegionServerInfo(final ServerName sn) {
148 return regionServers.get(sn);
149 }
150
151
152
153
154
155 public List<ServerName> getOnlineServers() {
156 synchronized (this.regionServers) {
157 return new ArrayList<ServerName>(this.regionServers.keySet());
158 }
159 }
160 }