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 org.apache.commons.logging.Log;
22 import org.apache.commons.logging.LogFactory;
23 import org.apache.hadoop.classification.InterfaceAudience;
24 import org.apache.hadoop.classification.InterfaceStability;
25 import org.apache.hadoop.conf.Configuration;
26 import org.apache.hadoop.hbase.Abortable;
27 import org.apache.hadoop.hbase.HConstants;
28 import org.apache.hadoop.hbase.exceptions.ZooKeeperConnectionException;
29 import org.apache.hadoop.hbase.util.Threads;
30 import org.apache.zookeeper.KeeperException;
31 import org.apache.zookeeper.WatchedEvent;
32 import org.apache.zookeeper.Watcher;
33 import org.apache.zookeeper.ZooDefs;
34 import org.apache.zookeeper.data.ACL;
35
36 import java.io.Closeable;
37 import java.io.IOException;
38 import java.util.ArrayList;
39 import java.util.List;
40 import java.util.concurrent.CopyOnWriteArrayList;
41 import java.util.concurrent.CountDownLatch;
42
43
44
45
46
47
48
49
50
51
52
53
54 @InterfaceAudience.Public
55 @InterfaceStability.Evolving
56 public class ZooKeeperWatcher implements Watcher, Abortable, Closeable {
57 private static final Log LOG = LogFactory.getLog(ZooKeeperWatcher.class);
58
59
60
61 private String identifier;
62
63
64 private String quorum;
65
66
67 private RecoverableZooKeeper recoverableZooKeeper;
68
69
70 protected Abortable abortable;
71
72
73 private final List<ZooKeeperListener> listeners =
74 new CopyOnWriteArrayList<ZooKeeperListener>();
75
76
77
78 public CountDownLatch saslLatch = new CountDownLatch(1);
79
80
81
82
83 public String baseZNode;
84
85 public String metaServerZNode;
86
87 public String rsZNode;
88
89 public String drainingZNode;
90
91 private String masterAddressZNode;
92
93 public String backupMasterAddressesZNode;
94
95 public String clusterStateZNode;
96
97 public String assignmentZNode;
98
99 public String tableZNode;
100
101 public String clusterIdZNode;
102
103 public String splitLogZNode;
104
105 public String balancerZNode;
106
107 public String tableLockZNode;
108
109
110 public static final ArrayList<ACL> CREATOR_ALL_AND_WORLD_READABLE =
111 new ArrayList<ACL>() { {
112 add(new ACL(ZooDefs.Perms.READ,ZooDefs.Ids.ANYONE_ID_UNSAFE));
113 add(new ACL(ZooDefs.Perms.ALL,ZooDefs.Ids.AUTH_IDS));
114 }};
115
116 private final Configuration conf;
117
118 private final Exception constructorCaller;
119
120
121
122
123
124
125
126
127 public ZooKeeperWatcher(Configuration conf, String identifier,
128 Abortable abortable) throws ZooKeeperConnectionException, IOException {
129 this(conf, identifier, abortable, false);
130 }
131
132
133
134
135
136
137
138 public ZooKeeperWatcher(Configuration conf, String identifier,
139 Abortable abortable, boolean canCreateBaseZNode)
140 throws IOException, ZooKeeperConnectionException {
141 this.conf = conf;
142
143
144 try {
145 throw new Exception("ZKW CONSTRUCTOR STACK TRACE FOR DEBUGGING");
146 } catch (Exception e) {
147 this.constructorCaller = e;
148 }
149 this.quorum = ZKConfig.getZKQuorumServersString(conf);
150
151
152 this.identifier = identifier;
153 this.abortable = abortable;
154 setNodeNames(conf);
155 this.recoverableZooKeeper = ZKUtil.connect(conf, quorum, this, identifier);
156 if (canCreateBaseZNode) {
157 createBaseZNodes();
158 }
159 }
160
161 private void createBaseZNodes() throws ZooKeeperConnectionException {
162 try {
163
164 ZKUtil.createWithParents(this, baseZNode);
165 ZKUtil.createAndFailSilent(this, assignmentZNode);
166 ZKUtil.createAndFailSilent(this, rsZNode);
167 ZKUtil.createAndFailSilent(this, drainingZNode);
168 ZKUtil.createAndFailSilent(this, tableZNode);
169 ZKUtil.createAndFailSilent(this, splitLogZNode);
170 ZKUtil.createAndFailSilent(this, backupMasterAddressesZNode);
171 ZKUtil.createAndFailSilent(this, tableLockZNode);
172 } catch (KeeperException e) {
173 throw new ZooKeeperConnectionException(
174 prefix("Unexpected KeeperException creating base node"), e);
175 }
176 }
177
178 @Override
179 public String toString() {
180 return this.identifier;
181 }
182
183
184
185
186
187
188
189 public String prefix(final String str) {
190 return this.toString() + " " + str;
191 }
192
193
194
195
196 private void setNodeNames(Configuration conf) {
197 baseZNode = conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
198 HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
199 metaServerZNode = ZKUtil.joinZNode(baseZNode,
200 conf.get("zookeeper.znode.metaserver", "meta-region-server"));
201 rsZNode = ZKUtil.joinZNode(baseZNode,
202 conf.get("zookeeper.znode.rs", "rs"));
203 drainingZNode = ZKUtil.joinZNode(baseZNode,
204 conf.get("zookeeper.znode.draining.rs", "draining"));
205 masterAddressZNode = ZKUtil.joinZNode(baseZNode,
206 conf.get("zookeeper.znode.master", "master"));
207 backupMasterAddressesZNode = ZKUtil.joinZNode(baseZNode,
208 conf.get("zookeeper.znode.backup.masters", "backup-masters"));
209 clusterStateZNode = ZKUtil.joinZNode(baseZNode,
210 conf.get("zookeeper.znode.state", "running"));
211 assignmentZNode = ZKUtil.joinZNode(baseZNode,
212 conf.get("zookeeper.znode.unassigned", "region-in-transition"));
213 tableZNode = ZKUtil.joinZNode(baseZNode,
214 conf.get("zookeeper.znode.tableEnableDisable", "table"));
215 clusterIdZNode = ZKUtil.joinZNode(baseZNode,
216 conf.get("zookeeper.znode.clusterId", "hbaseid"));
217 splitLogZNode = ZKUtil.joinZNode(baseZNode,
218 conf.get("zookeeper.znode.splitlog", HConstants.SPLIT_LOGDIR_NAME));
219 balancerZNode = ZKUtil.joinZNode(baseZNode,
220 conf.get("zookeeper.znode.balancer", "balancer"));
221 tableLockZNode = ZKUtil.joinZNode(baseZNode,
222 conf.get("zookeeper.znode.tableLock", "table-lock"));
223 }
224
225
226
227
228
229 public void registerListener(ZooKeeperListener listener) {
230 listeners.add(listener);
231 }
232
233
234
235
236
237
238 public void registerListenerFirst(ZooKeeperListener listener) {
239 listeners.add(0, listener);
240 }
241
242 public void unregisterListener(ZooKeeperListener listener) {
243 listeners.remove(listener);
244 }
245
246
247
248
249
250 public RecoverableZooKeeper getRecoverableZooKeeper() {
251 return recoverableZooKeeper;
252 }
253
254 public void reconnectAfterExpiration() throws IOException, InterruptedException {
255 recoverableZooKeeper.reconnectAfterExpiration();
256 }
257
258
259
260
261
262 public String getQuorum() {
263 return quorum;
264 }
265
266
267
268
269
270
271
272 @Override
273 public void process(WatchedEvent event) {
274 LOG.debug(prefix("Received ZooKeeper Event, " +
275 "type=" + event.getType() + ", " +
276 "state=" + event.getState() + ", " +
277 "path=" + event.getPath()));
278
279 switch(event.getType()) {
280
281
282 case None: {
283 connectionEvent(event);
284 break;
285 }
286
287
288
289 case NodeCreated: {
290 for(ZooKeeperListener listener : listeners) {
291 listener.nodeCreated(event.getPath());
292 }
293 break;
294 }
295
296 case NodeDeleted: {
297 for(ZooKeeperListener listener : listeners) {
298 listener.nodeDeleted(event.getPath());
299 }
300 break;
301 }
302
303 case NodeDataChanged: {
304 for(ZooKeeperListener listener : listeners) {
305 listener.nodeDataChanged(event.getPath());
306 }
307 break;
308 }
309
310 case NodeChildrenChanged: {
311 for(ZooKeeperListener listener : listeners) {
312 listener.nodeChildrenChanged(event.getPath());
313 }
314 break;
315 }
316 }
317 }
318
319
320
321
322
323
324
325
326
327
328
329
330
331 private void connectionEvent(WatchedEvent event) {
332 switch(event.getState()) {
333 case SyncConnected:
334
335
336 long finished = System.currentTimeMillis() +
337 this.conf.getLong("hbase.zookeeper.watcher.sync.connected.wait", 2000);
338 while (System.currentTimeMillis() < finished) {
339 Threads.sleep(1);
340 if (this.recoverableZooKeeper != null) break;
341 }
342 if (this.recoverableZooKeeper == null) {
343 LOG.error("ZK is null on connection event -- see stack trace " +
344 "for the stack trace when constructor was called on this zkw",
345 this.constructorCaller);
346 throw new NullPointerException("ZK is null");
347 }
348 this.identifier = this.identifier + "-0x" +
349 Long.toHexString(this.recoverableZooKeeper.getSessionId());
350
351 LOG.debug(this.identifier + " connected");
352 break;
353
354
355 case Disconnected:
356 LOG.debug(prefix("Received Disconnected from ZooKeeper, ignoring"));
357 break;
358
359 case Expired:
360 String msg = prefix(this.identifier + " received expired from " +
361 "ZooKeeper, aborting");
362
363
364 if (this.abortable != null) this.abortable.abort(msg,
365 new KeeperException.SessionExpiredException());
366 break;
367
368 case ConnectedReadOnly:
369 break;
370
371 default:
372 throw new IllegalStateException("Received event is not valid.");
373 }
374 }
375
376
377
378
379
380
381
382
383
384
385
386
387
388 public void sync(String path) {
389 this.recoverableZooKeeper.sync(path, null, null);
390 }
391
392
393
394
395
396
397
398
399
400
401
402 public void keeperException(KeeperException ke)
403 throws KeeperException {
404 LOG.error(prefix("Received unexpected KeeperException, re-throwing exception"), ke);
405 throw ke;
406 }
407
408
409
410
411
412
413
414
415
416
417
418
419 public void interruptedException(InterruptedException ie) {
420 LOG.debug(prefix("Received InterruptedException, doing nothing here"), ie);
421
422 Thread.currentThread().interrupt();
423
424 }
425
426
427
428
429
430
431 public void close() {
432 try {
433 if (recoverableZooKeeper != null) {
434 recoverableZooKeeper.close();
435 }
436 } catch (InterruptedException e) {
437 Thread.currentThread().interrupt();
438 }
439 }
440
441 public Configuration getConfiguration() {
442 return conf;
443 }
444
445 @Override
446 public void abort(String why, Throwable e) {
447 this.abortable.abort(why, e);
448 }
449
450 @Override
451 public boolean isAborted() {
452 return this.abortable.isAborted();
453 }
454
455
456
457
458 public String getMasterAddressZNode() {
459 return this.masterAddressZNode;
460 }
461
462 }