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.BufferedReader;
22 import java.io.IOException;
23 import java.io.InputStreamReader;
24 import java.io.PrintWriter;
25 import java.net.InetSocketAddress;
26 import java.net.Socket;
27 import java.util.ArrayList;
28 import java.util.Arrays;
29 import java.util.Deque;
30 import java.util.HashMap;
31 import java.util.LinkedList;
32 import java.util.List;
33 import java.util.Map;
34
35 import javax.security.auth.login.AppConfigurationEntry;
36 import javax.security.auth.login.AppConfigurationEntry.LoginModuleControlFlag;
37
38 import org.apache.commons.lang.StringUtils;
39 import org.apache.commons.logging.Log;
40 import org.apache.commons.logging.LogFactory;
41 import org.apache.hadoop.conf.Configuration;
42 import org.apache.hadoop.hbase.AuthUtil;
43 import org.apache.hadoop.hbase.HBaseConfiguration;
44 import org.apache.hadoop.hbase.HConstants;
45 import org.apache.hadoop.hbase.ServerName;
46 import org.apache.hadoop.hbase.classification.InterfaceAudience;
47 import org.apache.hadoop.hbase.exceptions.DeserializationException;
48 import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
49 import org.apache.hadoop.hbase.protobuf.generated.ClusterStatusProtos;
50 import org.apache.hadoop.hbase.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds;
51 import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
52 import org.apache.hadoop.hbase.replication.ReplicationStateZKBase;
53 import org.apache.hadoop.hbase.security.Superusers;
54 import org.apache.hadoop.hbase.util.ByteStringer;
55 import org.apache.hadoop.hbase.util.Bytes;
56 import org.apache.hadoop.hbase.util.Threads;
57 import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp.CreateAndFailSilent;
58 import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp.DeleteNodeFailSilent;
59 import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp.SetData;
60 import org.apache.hadoop.security.SecurityUtil;
61 import org.apache.hadoop.security.authentication.util.KerberosUtil;
62 import org.apache.zookeeper.AsyncCallback;
63 import org.apache.zookeeper.CreateMode;
64 import org.apache.zookeeper.KeeperException;
65 import org.apache.zookeeper.KeeperException.NoNodeException;
66 import org.apache.zookeeper.Op;
67 import org.apache.zookeeper.Watcher;
68 import org.apache.zookeeper.ZooDefs.Ids;
69 import org.apache.zookeeper.ZooDefs.Perms;
70 import org.apache.zookeeper.ZooKeeper;
71 import org.apache.zookeeper.client.ZooKeeperSaslClient;
72 import org.apache.zookeeper.data.ACL;
73 import org.apache.zookeeper.data.Id;
74 import org.apache.zookeeper.data.Stat;
75 import org.apache.zookeeper.proto.CreateRequest;
76 import org.apache.zookeeper.proto.DeleteRequest;
77 import org.apache.zookeeper.proto.SetDataRequest;
78 import org.apache.zookeeper.server.ZooKeeperSaslServer;
79
80 import com.google.protobuf.InvalidProtocolBufferException;
81
82
83
84
85
86
87
88
89
90
91 @InterfaceAudience.Private
92 public class ZKUtil {
93 private static final Log LOG = LogFactory.getLog(ZKUtil.class);
94
95
96 public static final char ZNODE_PATH_SEPARATOR = '/';
97 private static int zkDumpConnectionTimeOut;
98
99
100
101
102
103
104
105
106
107
108
109
110 public static RecoverableZooKeeper connect(Configuration conf, Watcher watcher)
111 throws IOException {
112 String ensemble = ZKConfig.getZKQuorumServersString(conf);
113 return connect(conf, ensemble, watcher);
114 }
115
116 public static RecoverableZooKeeper connect(Configuration conf, String ensemble,
117 Watcher watcher)
118 throws IOException {
119 return connect(conf, ensemble, watcher, null);
120 }
121
122 public static RecoverableZooKeeper connect(Configuration conf, String ensemble,
123 Watcher watcher, final String identifier)
124 throws IOException {
125 if(ensemble == null) {
126 throw new IOException("Unable to determine ZooKeeper ensemble");
127 }
128 int timeout = conf.getInt(HConstants.ZK_SESSION_TIMEOUT,
129 HConstants.DEFAULT_ZK_SESSION_TIMEOUT);
130 if (LOG.isTraceEnabled()) {
131 LOG.trace(identifier + " opening connection to ZooKeeper ensemble=" + ensemble);
132 }
133 int retry = conf.getInt("zookeeper.recovery.retry", 3);
134 int retryIntervalMillis =
135 conf.getInt("zookeeper.recovery.retry.intervalmill", 1000);
136 zkDumpConnectionTimeOut = conf.getInt("zookeeper.dump.connection.timeout",
137 1000);
138 return new RecoverableZooKeeper(ensemble, timeout, watcher,
139 retry, retryIntervalMillis, identifier);
140 }
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156 public static void loginServer(Configuration conf, String keytabFileKey,
157 String userNameKey, String hostname) throws IOException {
158 login(conf, keytabFileKey, userNameKey, hostname,
159 ZooKeeperSaslServer.LOGIN_CONTEXT_NAME_KEY,
160 JaasConfiguration.SERVER_KEYTAB_KERBEROS_CONFIG_NAME);
161 }
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177 public static void loginClient(Configuration conf, String keytabFileKey,
178 String userNameKey, String hostname) throws IOException {
179 login(conf, keytabFileKey, userNameKey, hostname,
180 ZooKeeperSaslClient.LOGIN_CONTEXT_NAME_KEY,
181 JaasConfiguration.CLIENT_KEYTAB_KERBEROS_CONFIG_NAME);
182 }
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200 private static void login(Configuration conf, String keytabFileKey,
201 String userNameKey, String hostname,
202 String loginContextProperty, String loginContextName)
203 throws IOException {
204 if (!isSecureZooKeeper(conf))
205 return;
206
207
208
209 if (System.getProperty("java.security.auth.login.config") != null)
210 return;
211
212
213 String keytabFilename = conf.get(keytabFileKey);
214 if (keytabFilename == null) {
215 LOG.warn("no keytab specified for: " + keytabFileKey);
216 return;
217 }
218
219 String principalConfig = conf.get(userNameKey, System.getProperty("user.name"));
220 String principalName = SecurityUtil.getServerPrincipal(principalConfig, hostname);
221
222
223
224
225 JaasConfiguration jaasConf = new JaasConfiguration(loginContextName,
226 principalName, keytabFilename);
227 javax.security.auth.login.Configuration.setConfiguration(jaasConf);
228 System.setProperty(loginContextProperty, loginContextName);
229 }
230
231
232
233
234 private static class JaasConfiguration extends javax.security.auth.login.Configuration {
235 private static final String SERVER_KEYTAB_KERBEROS_CONFIG_NAME =
236 "zookeeper-server-keytab-kerberos";
237 private static final String CLIENT_KEYTAB_KERBEROS_CONFIG_NAME =
238 "zookeeper-client-keytab-kerberos";
239
240 private static final Map<String, String> BASIC_JAAS_OPTIONS =
241 new HashMap<String,String>();
242 static {
243 String jaasEnvVar = System.getenv("HBASE_JAAS_DEBUG");
244 if (jaasEnvVar != null && "true".equalsIgnoreCase(jaasEnvVar)) {
245 BASIC_JAAS_OPTIONS.put("debug", "true");
246 }
247 }
248
249 private static final Map<String,String> KEYTAB_KERBEROS_OPTIONS =
250 new HashMap<String,String>();
251 static {
252 KEYTAB_KERBEROS_OPTIONS.put("doNotPrompt", "true");
253 KEYTAB_KERBEROS_OPTIONS.put("storeKey", "true");
254 KEYTAB_KERBEROS_OPTIONS.put("refreshKrb5Config", "true");
255 KEYTAB_KERBEROS_OPTIONS.putAll(BASIC_JAAS_OPTIONS);
256 }
257
258 private static final AppConfigurationEntry KEYTAB_KERBEROS_LOGIN =
259 new AppConfigurationEntry(KerberosUtil.getKrb5LoginModuleName(),
260 LoginModuleControlFlag.REQUIRED,
261 KEYTAB_KERBEROS_OPTIONS);
262
263 private static final AppConfigurationEntry[] KEYTAB_KERBEROS_CONF =
264 new AppConfigurationEntry[]{KEYTAB_KERBEROS_LOGIN};
265
266 private javax.security.auth.login.Configuration baseConfig;
267 private final String loginContextName;
268 private final boolean useTicketCache;
269 private final String keytabFile;
270 private final String principal;
271
272 public JaasConfiguration(String loginContextName, String principal) {
273 this(loginContextName, principal, null, true);
274 }
275
276 public JaasConfiguration(String loginContextName, String principal, String keytabFile) {
277 this(loginContextName, principal, keytabFile, keytabFile == null || keytabFile.length() == 0);
278 }
279
280 private JaasConfiguration(String loginContextName, String principal,
281 String keytabFile, boolean useTicketCache) {
282 try {
283 this.baseConfig = javax.security.auth.login.Configuration.getConfiguration();
284 } catch (SecurityException e) {
285 this.baseConfig = null;
286 }
287 this.loginContextName = loginContextName;
288 this.useTicketCache = useTicketCache;
289 this.keytabFile = keytabFile;
290 this.principal = principal;
291 LOG.info("JaasConfiguration loginContextName=" + loginContextName +
292 " principal=" + principal + " useTicketCache=" + useTicketCache +
293 " keytabFile=" + keytabFile);
294 }
295
296 @Override
297 public AppConfigurationEntry[] getAppConfigurationEntry(String appName) {
298 if (loginContextName.equals(appName)) {
299 if (!useTicketCache) {
300 KEYTAB_KERBEROS_OPTIONS.put("keyTab", keytabFile);
301 KEYTAB_KERBEROS_OPTIONS.put("useKeyTab", "true");
302 }
303 KEYTAB_KERBEROS_OPTIONS.put("principal", principal);
304 KEYTAB_KERBEROS_OPTIONS.put("useTicketCache", useTicketCache ? "true" : "false");
305 return KEYTAB_KERBEROS_CONF;
306 }
307 if (baseConfig != null) return baseConfig.getAppConfigurationEntry(appName);
308 return(null);
309 }
310 }
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326 public static String joinZNode(String prefix, String suffix) {
327 return prefix + ZNODE_PATH_SEPARATOR + suffix;
328 }
329
330
331
332
333
334
335 public static String getParent(String node) {
336 int idx = node.lastIndexOf(ZNODE_PATH_SEPARATOR);
337 return idx <= 0 ? null : node.substring(0, idx);
338 }
339
340
341
342
343
344
345 public static String getNodeName(String path) {
346 return path.substring(path.lastIndexOf("/")+1);
347 }
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363 public static boolean watchAndCheckExists(ZooKeeperWatcher zkw, String znode)
364 throws KeeperException {
365 try {
366 Stat s = zkw.getRecoverableZooKeeper().exists(znode, zkw);
367 boolean exists = s != null ? true : false;
368 if (exists) {
369 LOG.debug(zkw.prefix("Set watcher on existing znode=" + znode));
370 } else {
371 LOG.debug(zkw.prefix("Set watcher on znode that does not yet exist, " + znode));
372 }
373 return exists;
374 } catch (KeeperException e) {
375 LOG.warn(zkw.prefix("Unable to set watcher on znode " + znode), e);
376 zkw.keeperException(e);
377 return false;
378 } catch (InterruptedException e) {
379 LOG.warn(zkw.prefix("Unable to set watcher on znode " + znode), e);
380 zkw.interruptedException(e);
381 return false;
382 }
383 }
384
385
386
387
388
389
390
391
392
393
394
395 public static boolean setWatchIfNodeExists(ZooKeeperWatcher zkw, String znode)
396 throws KeeperException {
397 try {
398 zkw.getRecoverableZooKeeper().getData(znode, true, null);
399 return true;
400 } catch (NoNodeException e) {
401 return false;
402 } catch (InterruptedException e) {
403 LOG.warn(zkw.prefix("Unable to set watcher on znode " + znode), e);
404 zkw.interruptedException(e);
405 return false;
406 }
407 }
408
409
410
411
412
413
414
415
416
417 public static int checkExists(ZooKeeperWatcher zkw, String znode)
418 throws KeeperException {
419 try {
420 Stat s = zkw.getRecoverableZooKeeper().exists(znode, null);
421 return s != null ? s.getVersion() : -1;
422 } catch (KeeperException e) {
423 LOG.warn(zkw.prefix("Unable to set watcher on znode (" + znode + ")"), e);
424 zkw.keeperException(e);
425 return -1;
426 } catch (InterruptedException e) {
427 LOG.warn(zkw.prefix("Unable to set watcher on znode (" + znode + ")"), e);
428 zkw.interruptedException(e);
429 return -1;
430 }
431 }
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453 public static List<String> listChildrenAndWatchForNewChildren(
454 ZooKeeperWatcher zkw, String znode)
455 throws KeeperException {
456 try {
457 List<String> children = zkw.getRecoverableZooKeeper().getChildren(znode, zkw);
458 return children;
459 } catch(KeeperException.NoNodeException ke) {
460 LOG.debug(zkw.prefix("Unable to list children of znode " + znode + " " +
461 "because node does not exist (not an error)"));
462 return null;
463 } catch (KeeperException e) {
464 LOG.warn(zkw.prefix("Unable to list children of znode " + znode + " "), e);
465 zkw.keeperException(e);
466 return null;
467 } catch (InterruptedException e) {
468 LOG.warn(zkw.prefix("Unable to list children of znode " + znode + " "), e);
469 zkw.interruptedException(e);
470 return null;
471 }
472 }
473
474
475
476
477
478
479
480
481
482
483 public static List<String> listChildrenAndWatchThem(ZooKeeperWatcher zkw,
484 String znode) throws KeeperException {
485 List<String> children = listChildrenAndWatchForNewChildren(zkw, znode);
486 if (children == null) {
487 return null;
488 }
489 for (String child : children) {
490 watchAndCheckExists(zkw, joinZNode(znode, child));
491 }
492 return children;
493 }
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509 public static List<String> listChildrenNoWatch(ZooKeeperWatcher zkw, String znode)
510 throws KeeperException {
511 List<String> children = null;
512 try {
513
514 children = zkw.getRecoverableZooKeeper().getChildren(znode, null);
515 } catch(KeeperException.NoNodeException nne) {
516 return null;
517 } catch(InterruptedException ie) {
518 zkw.interruptedException(ie);
519 }
520 return children;
521 }
522
523
524
525
526
527 @Deprecated
528 public static class NodeAndData {
529 private String node;
530 private byte [] data;
531 public NodeAndData(String node, byte [] data) {
532 this.node = node;
533 this.data = data;
534 }
535 public String getNode() {
536 return node;
537 }
538 public byte [] getData() {
539 return data;
540 }
541 @Override
542 public String toString() {
543 return node;
544 }
545 public boolean isEmpty() {
546 return (data == null || data.length == 0);
547 }
548 }
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566 public static boolean nodeHasChildren(ZooKeeperWatcher zkw, String znode)
567 throws KeeperException {
568 try {
569 return !zkw.getRecoverableZooKeeper().getChildren(znode, null).isEmpty();
570 } catch(KeeperException.NoNodeException ke) {
571 LOG.debug(zkw.prefix("Unable to list children of znode " + znode + " " +
572 "because node does not exist (not an error)"));
573 return false;
574 } catch (KeeperException e) {
575 LOG.warn(zkw.prefix("Unable to list children of znode " + znode), e);
576 zkw.keeperException(e);
577 return false;
578 } catch (InterruptedException e) {
579 LOG.warn(zkw.prefix("Unable to list children of znode " + znode), e);
580 zkw.interruptedException(e);
581 return false;
582 }
583 }
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598 public static int getNumberOfChildren(ZooKeeperWatcher zkw, String znode)
599 throws KeeperException {
600 try {
601 Stat stat = zkw.getRecoverableZooKeeper().exists(znode, null);
602 return stat == null ? 0 : stat.getNumChildren();
603 } catch(KeeperException e) {
604 LOG.warn(zkw.prefix("Unable to get children of node " + znode));
605 zkw.keeperException(e);
606 } catch(InterruptedException e) {
607 zkw.interruptedException(e);
608 }
609 return 0;
610 }
611
612
613
614
615
616
617
618
619
620
621 public static byte [] getData(ZooKeeperWatcher zkw, String znode)
622 throws KeeperException, InterruptedException {
623 try {
624 byte [] data = zkw.getRecoverableZooKeeper().getData(znode, null, null);
625 logRetrievedMsg(zkw, znode, data, false);
626 return data;
627 } catch (KeeperException.NoNodeException e) {
628 LOG.debug(zkw.prefix("Unable to get data of znode " + znode + " " +
629 "because node does not exist (not an error)"));
630 return null;
631 } catch (KeeperException e) {
632 LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
633 zkw.keeperException(e);
634 return null;
635 }
636 }
637
638
639
640
641
642
643
644
645
646
647
648
649 public static byte [] getDataAndWatch(ZooKeeperWatcher zkw, String znode)
650 throws KeeperException {
651 return getDataInternal(zkw, znode, null, true);
652 }
653
654
655
656
657
658
659
660
661
662
663
664
665
666 public static byte[] getDataAndWatch(ZooKeeperWatcher zkw, String znode,
667 Stat stat) throws KeeperException {
668 return getDataInternal(zkw, znode, stat, true);
669 }
670
671 private static byte[] getDataInternal(ZooKeeperWatcher zkw, String znode, Stat stat,
672 boolean watcherSet)
673 throws KeeperException {
674 try {
675 byte [] data = zkw.getRecoverableZooKeeper().getData(znode, zkw, stat);
676 logRetrievedMsg(zkw, znode, data, watcherSet);
677 return data;
678 } catch (KeeperException.NoNodeException e) {
679
680
681 LOG.trace(zkw.prefix("Unable to get data of znode " + znode + " " +
682 "because node does not exist (not an error)"));
683 return null;
684 } catch (KeeperException e) {
685 LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
686 zkw.keeperException(e);
687 return null;
688 } catch (InterruptedException e) {
689 LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
690 zkw.interruptedException(e);
691 return null;
692 }
693 }
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710 public static byte [] getDataNoWatch(ZooKeeperWatcher zkw, String znode,
711 Stat stat)
712 throws KeeperException {
713 try {
714 byte [] data = zkw.getRecoverableZooKeeper().getData(znode, null, stat);
715 logRetrievedMsg(zkw, znode, data, false);
716 return data;
717 } catch (KeeperException.NoNodeException e) {
718 LOG.debug(zkw.prefix("Unable to get data of znode " + znode + " " +
719 "because node does not exist (not necessarily an error)"));
720 return null;
721 } catch (KeeperException e) {
722 LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
723 zkw.keeperException(e);
724 return null;
725 } catch (InterruptedException e) {
726 LOG.warn(zkw.prefix("Unable to get data of znode " + znode), e);
727 zkw.interruptedException(e);
728 return null;
729 }
730 }
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749 public static List<NodeAndData> getChildDataAndWatchForNewChildren(
750 ZooKeeperWatcher zkw, String baseNode) throws KeeperException {
751 List<String> nodes =
752 ZKUtil.listChildrenAndWatchForNewChildren(zkw, baseNode);
753 if (nodes != null) {
754 List<NodeAndData> newNodes = new ArrayList<NodeAndData>();
755 for (String node : nodes) {
756 String nodePath = ZKUtil.joinZNode(baseNode, node);
757 byte[] data = ZKUtil.getDataAndWatch(zkw, nodePath);
758 newNodes.add(new NodeAndData(nodePath, data));
759 }
760 return newNodes;
761 }
762 return null;
763 }
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781 public static void updateExistingNodeData(ZooKeeperWatcher zkw, String znode,
782 byte [] data, int expectedVersion)
783 throws KeeperException {
784 try {
785 zkw.getRecoverableZooKeeper().setData(znode, data, expectedVersion);
786 } catch(InterruptedException ie) {
787 zkw.interruptedException(ie);
788 }
789 }
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815 public static boolean setData(ZooKeeperWatcher zkw, String znode,
816 byte [] data, int expectedVersion)
817 throws KeeperException, KeeperException.NoNodeException {
818 try {
819 return zkw.getRecoverableZooKeeper().setData(znode, data, expectedVersion) != null;
820 } catch (InterruptedException e) {
821 zkw.interruptedException(e);
822 return false;
823 }
824 }
825
826
827
828
829
830
831
832
833
834
835 public static void createSetData(final ZooKeeperWatcher zkw, final String znode,
836 final byte [] data)
837 throws KeeperException {
838 if (checkExists(zkw, znode) == -1) {
839 ZKUtil.createWithParents(zkw, znode, data);
840 } else {
841 ZKUtil.setData(zkw, znode, data);
842 }
843 }
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861 public static void setData(ZooKeeperWatcher zkw, String znode, byte [] data)
862 throws KeeperException, KeeperException.NoNodeException {
863 setData(zkw, (SetData)ZKUtilOp.setData(znode, data));
864 }
865
866 private static void setData(ZooKeeperWatcher zkw, SetData setData)
867 throws KeeperException, KeeperException.NoNodeException {
868 SetDataRequest sd = (SetDataRequest)toZooKeeperOp(zkw, setData).toRequestRecord();
869 setData(zkw, sd.getPath(), sd.getData(), sd.getVersion());
870 }
871
872
873
874
875
876
877 public static boolean isSecureZooKeeper(Configuration conf) {
878
879
880 try {
881 javax.security.auth.login.Configuration testConfig =
882 javax.security.auth.login.Configuration.getConfiguration();
883 if (testConfig.getAppConfigurationEntry("Client") == null
884 && testConfig.getAppConfigurationEntry(
885 JaasConfiguration.CLIENT_KEYTAB_KERBEROS_CONFIG_NAME) == null
886 && testConfig.getAppConfigurationEntry(
887 JaasConfiguration.SERVER_KEYTAB_KERBEROS_CONFIG_NAME) == null) {
888 return false;
889 }
890 } catch(Exception e) {
891
892 return false;
893 }
894
895
896 return "kerberos".equalsIgnoreCase(conf.get("hbase.security.authentication"));
897 }
898
899 private static ArrayList<ACL> createACL(ZooKeeperWatcher zkw, String node) {
900 return createACL(zkw, node, isSecureZooKeeper(zkw.getConfiguration()));
901 }
902
903 public static ArrayList<ACL> createACL(ZooKeeperWatcher zkw, String node,
904 boolean isSecureZooKeeper) {
905 if (!node.startsWith(zkw.baseZNode)) {
906 return Ids.OPEN_ACL_UNSAFE;
907 }
908 if (isSecureZooKeeper) {
909 ArrayList<ACL> acls = new ArrayList<ACL>();
910
911 String[] superUsers = zkw.getConfiguration().getStrings(Superusers.SUPERUSER_CONF_KEY);
912 if (superUsers != null) {
913 List<String> groups = new ArrayList<String>();
914 for (String user : superUsers) {
915 if (user.startsWith(AuthUtil.GROUP_PREFIX)) {
916
917 groups.add(user);
918 } else {
919 acls.add(new ACL(Perms.ALL, new Id("sasl", user)));
920 }
921 }
922 if (!groups.isEmpty()) {
923 LOG.warn("Znode ACL setting for group " + groups
924 + " is skipped, Zookeeper doesn't support this feature presently.");
925 }
926 }
927
928
929 if (zkw.isClientReadable(node)) {
930 acls.addAll(Ids.CREATOR_ALL_ACL);
931 acls.addAll(Ids.READ_ACL_UNSAFE);
932 } else {
933 acls.addAll(Ids.CREATOR_ALL_ACL);
934 }
935 return acls;
936 } else {
937 return Ids.OPEN_ACL_UNSAFE;
938 }
939 }
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963 public static boolean createEphemeralNodeAndWatch(ZooKeeperWatcher zkw,
964 String znode, byte [] data)
965 throws KeeperException {
966 boolean ret = true;
967 try {
968 zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
969 CreateMode.EPHEMERAL);
970 } catch (KeeperException.NodeExistsException nee) {
971 ret = false;
972 } catch (InterruptedException e) {
973 LOG.info("Interrupted", e);
974 Thread.currentThread().interrupt();
975 }
976 if(!watchAndCheckExists(zkw, znode)) {
977
978 return createEphemeralNodeAndWatch(zkw, znode, data);
979 }
980 return ret;
981 }
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
1001
1002
1003 public static boolean createNodeIfNotExistsAndWatch(
1004 ZooKeeperWatcher zkw, String znode, byte [] data)
1005 throws KeeperException {
1006 boolean ret = true;
1007 try {
1008 zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
1009 CreateMode.PERSISTENT);
1010 } catch (KeeperException.NodeExistsException nee) {
1011 ret = false;
1012 } catch (InterruptedException e) {
1013 zkw.interruptedException(e);
1014 return false;
1015 }
1016 try {
1017 zkw.getRecoverableZooKeeper().exists(znode, zkw);
1018 } catch (InterruptedException e) {
1019 zkw.interruptedException(e);
1020 return false;
1021 }
1022 return ret;
1023 }
1024
1025
1026
1027
1028
1029
1030
1031
1032
1033
1034
1035
1036
1037
1038
1039 public static String createNodeIfNotExistsNoWatch(ZooKeeperWatcher zkw, String znode,
1040 byte[] data, CreateMode createMode) throws KeeperException {
1041
1042 String createdZNode = null;
1043 try {
1044 createdZNode = zkw.getRecoverableZooKeeper().create(znode, data,
1045 createACL(zkw, znode), createMode);
1046 } catch (KeeperException.NodeExistsException nee) {
1047 return znode;
1048 } catch (InterruptedException e) {
1049 zkw.interruptedException(e);
1050 return null;
1051 }
1052 return createdZNode;
1053 }
1054
1055
1056
1057
1058
1059
1060
1061
1062
1063
1064
1065
1066
1067
1068
1069
1070
1071 public static int createAndWatch(ZooKeeperWatcher zkw,
1072 String znode, byte [] data)
1073 throws KeeperException, KeeperException.NodeExistsException {
1074 try {
1075 zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
1076 CreateMode.PERSISTENT);
1077 Stat stat = zkw.getRecoverableZooKeeper().exists(znode, zkw);
1078 if (stat == null){
1079
1080 throw KeeperException.create(KeeperException.Code.SYSTEMERROR,
1081 "ZK.exists returned null (i.e.: znode does not exist) for znode=" + znode);
1082 }
1083 return stat.getVersion();
1084 } catch (InterruptedException e) {
1085 zkw.interruptedException(e);
1086 return -1;
1087 }
1088 }
1089
1090
1091
1092
1093
1094
1095
1096
1097
1098
1099
1100
1101
1102
1103
1104
1105 public static void asyncCreate(ZooKeeperWatcher zkw,
1106 String znode, byte [] data, final AsyncCallback.StringCallback cb,
1107 final Object ctx) {
1108 zkw.getRecoverableZooKeeper().getZooKeeper().create(znode, data,
1109 createACL(zkw, znode), CreateMode.PERSISTENT, cb, ctx);
1110 }
1111
1112
1113
1114
1115
1116
1117
1118
1119
1120
1121
1122 public static void createAndFailSilent(ZooKeeperWatcher zkw,
1123 String znode) throws KeeperException {
1124 createAndFailSilent(zkw, znode, new byte[0]);
1125 }
1126
1127
1128
1129
1130
1131
1132
1133
1134
1135
1136
1137
1138 public static void createAndFailSilent(ZooKeeperWatcher zkw,
1139 String znode, byte[] data)
1140 throws KeeperException {
1141 createAndFailSilent(zkw,
1142 (CreateAndFailSilent)ZKUtilOp.createAndFailSilent(znode, data));
1143 }
1144
1145 private static void createAndFailSilent(ZooKeeperWatcher zkw, CreateAndFailSilent cafs)
1146 throws KeeperException {
1147 CreateRequest create = (CreateRequest)toZooKeeperOp(zkw, cafs).toRequestRecord();
1148 String znode = create.getPath();
1149 try {
1150 RecoverableZooKeeper zk = zkw.getRecoverableZooKeeper();
1151 if (zk.exists(znode, false) == null) {
1152 zk.create(znode, create.getData(), create.getAcl(), CreateMode.fromFlag(create.getFlags()));
1153 }
1154 } catch(KeeperException.NodeExistsException nee) {
1155 } catch(KeeperException.NoAuthException nee){
1156 try {
1157 if (null == zkw.getRecoverableZooKeeper().exists(znode, false)) {
1158
1159 throw(nee);
1160 }
1161 } catch (InterruptedException ie) {
1162 zkw.interruptedException(ie);
1163 }
1164 } catch(InterruptedException ie) {
1165 zkw.interruptedException(ie);
1166 }
1167 }
1168
1169
1170
1171
1172
1173
1174
1175
1176
1177
1178
1179
1180 public static void createWithParents(ZooKeeperWatcher zkw, String znode)
1181 throws KeeperException {
1182 createWithParents(zkw, znode, new byte[0]);
1183 }
1184
1185
1186
1187
1188
1189
1190
1191
1192
1193
1194
1195
1196
1197
1198 public static void createWithParents(ZooKeeperWatcher zkw, String znode, byte[] data)
1199 throws KeeperException {
1200 try {
1201 if(znode == null) {
1202 return;
1203 }
1204 zkw.getRecoverableZooKeeper().create(znode, data, createACL(zkw, znode),
1205 CreateMode.PERSISTENT);
1206 } catch(KeeperException.NodeExistsException nee) {
1207 return;
1208 } catch(KeeperException.NoNodeException nne) {
1209 createWithParents(zkw, getParent(znode));
1210 createWithParents(zkw, znode, data);
1211 } catch(InterruptedException ie) {
1212 zkw.interruptedException(ie);
1213 }
1214 }
1215
1216
1217
1218
1219
1220
1221
1222
1223 public static void deleteNode(ZooKeeperWatcher zkw, String node)
1224 throws KeeperException {
1225 deleteNode(zkw, node, -1);
1226 }
1227
1228
1229
1230
1231
1232 public static boolean deleteNode(ZooKeeperWatcher zkw, String node,
1233 int version)
1234 throws KeeperException {
1235 try {
1236 zkw.getRecoverableZooKeeper().delete(node, version);
1237 return true;
1238 } catch(KeeperException.BadVersionException bve) {
1239 return false;
1240 } catch(InterruptedException ie) {
1241 zkw.interruptedException(ie);
1242 return false;
1243 }
1244 }
1245
1246
1247
1248
1249
1250
1251
1252 public static void deleteNodeFailSilent(ZooKeeperWatcher zkw, String node)
1253 throws KeeperException {
1254 deleteNodeFailSilent(zkw,
1255 (DeleteNodeFailSilent)ZKUtilOp.deleteNodeFailSilent(node));
1256 }
1257
1258 private static void deleteNodeFailSilent(ZooKeeperWatcher zkw,
1259 DeleteNodeFailSilent dnfs) throws KeeperException {
1260 DeleteRequest delete = (DeleteRequest)toZooKeeperOp(zkw, dnfs).toRequestRecord();
1261 try {
1262 zkw.getRecoverableZooKeeper().delete(delete.getPath(), delete.getVersion());
1263 } catch(KeeperException.NoNodeException nne) {
1264 } catch(InterruptedException ie) {
1265 zkw.interruptedException(ie);
1266 }
1267 }
1268
1269
1270
1271
1272
1273
1274
1275
1276
1277
1278 public static void deleteNodeRecursively(ZooKeeperWatcher zkw, String node)
1279 throws KeeperException {
1280 deleteNodeRecursivelyMultiOrSequential(zkw, true, node);
1281 }
1282
1283
1284
1285
1286
1287
1288
1289
1290
1291
1292
1293
1294 public static void deleteChildrenRecursively(ZooKeeperWatcher zkw, String node)
1295 throws KeeperException {
1296 deleteChildrenRecursivelyMultiOrSequential(zkw, true, node);
1297 }
1298
1299
1300
1301
1302
1303
1304
1305
1306
1307
1308
1309
1310
1311
1312
1313
1314
1315
1316
1317
1318
1319
1320
1321
1322
1323
1324
1325
1326
1327
1328
1329
1330
1331
1332
1333 public static void deleteChildrenRecursivelyMultiOrSequential(
1334 ZooKeeperWatcher zkw, boolean runSequentialOnMultiFailure,
1335 String... pathRoots) throws KeeperException {
1336 if (pathRoots == null || pathRoots.length <= 0) {
1337 LOG.warn("Given path is not valid!");
1338 return;
1339 }
1340 List<ZKUtilOp> ops = new ArrayList<ZKUtil.ZKUtilOp>();
1341 for (String eachRoot : pathRoots) {
1342 List<String> children = listChildrenBFSNoWatch(zkw, eachRoot);
1343
1344 for (int i = children.size() - 1; i >= 0; --i) {
1345 ops.add(ZKUtilOp.deleteNodeFailSilent(children.get(i)));
1346 }
1347 }
1348
1349 if (ops.size() > 0) {
1350 multiOrSequential(zkw, ops, runSequentialOnMultiFailure);
1351 }
1352 }
1353
1354
1355
1356
1357
1358
1359
1360
1361
1362
1363
1364
1365
1366
1367
1368
1369
1370
1371
1372
1373
1374
1375
1376
1377
1378
1379
1380
1381
1382
1383
1384
1385
1386
1387
1388 public static void deleteNodeRecursivelyMultiOrSequential(ZooKeeperWatcher zkw,
1389 boolean runSequentialOnMultiFailure, String... pathRoots) throws KeeperException {
1390 if (pathRoots == null || pathRoots.length <= 0) {
1391 LOG.warn("Given path is not valid!");
1392 return;
1393 }
1394 List<ZKUtilOp> ops = new ArrayList<ZKUtil.ZKUtilOp>();
1395 for (String eachRoot : pathRoots) {
1396
1397
1398 List<String> children = listChildrenBFSAndWatchThem(zkw, eachRoot);
1399
1400 for (int i = children.size() - 1; i >= 0; --i) {
1401 ops.add(ZKUtilOp.deleteNodeFailSilent(children.get(i)));
1402 }
1403 try {
1404 if (zkw.getRecoverableZooKeeper().exists(eachRoot, zkw) != null) {
1405 ops.add(ZKUtilOp.deleteNodeFailSilent(eachRoot));
1406 }
1407 } catch (InterruptedException e) {
1408 zkw.interruptedException(e);
1409 }
1410 }
1411
1412 if (ops.size() > 0) {
1413 multiOrSequential(zkw, ops, runSequentialOnMultiFailure);
1414 }
1415 }
1416
1417
1418
1419
1420
1421
1422
1423
1424
1425
1426
1427
1428
1429
1430 private static List<String> listChildrenBFSNoWatch(ZooKeeperWatcher zkw,
1431 final String znode) throws KeeperException {
1432 Deque<String> queue = new LinkedList<String>();
1433 List<String> tree = new ArrayList<String>();
1434 queue.add(znode);
1435 while (true) {
1436 String node = queue.pollFirst();
1437 if (node == null) {
1438 break;
1439 }
1440 List<String> children = listChildrenNoWatch(zkw, node);
1441 if (children == null) {
1442 continue;
1443 }
1444 for (final String child : children) {
1445 final String childPath = node + "/" + child;
1446 queue.add(childPath);
1447 tree.add(childPath);
1448 }
1449 }
1450 return tree;
1451 }
1452
1453
1454
1455
1456
1457
1458
1459
1460
1461
1462
1463
1464
1465
1466 private static List<String> listChildrenBFSAndWatchThem(ZooKeeperWatcher zkw, final String znode)
1467 throws KeeperException {
1468 Deque<String> queue = new LinkedList<String>();
1469 List<String> tree = new ArrayList<String>();
1470 queue.add(znode);
1471 while (true) {
1472 String node = queue.pollFirst();
1473 if (node == null) {
1474 break;
1475 }
1476 List<String> children = listChildrenAndWatchThem(zkw, node);
1477 if (children == null) {
1478 continue;
1479 }
1480 for (final String child : children) {
1481 final String childPath = node + "/" + child;
1482 queue.add(childPath);
1483 tree.add(childPath);
1484 }
1485 }
1486 return tree;
1487 }
1488
1489
1490
1491
1492
1493
1494 public abstract static class ZKUtilOp {
1495 private String path;
1496
1497 private ZKUtilOp(String path) {
1498 this.path = path;
1499 }
1500
1501
1502
1503
1504 public static ZKUtilOp createAndFailSilent(String path, byte[] data) {
1505 return new CreateAndFailSilent(path, data);
1506 }
1507
1508
1509
1510
1511 public static ZKUtilOp deleteNodeFailSilent(String path) {
1512 return new DeleteNodeFailSilent(path);
1513 }
1514
1515
1516
1517
1518 public static ZKUtilOp setData(String path, byte [] data) {
1519 return new SetData(path, data);
1520 }
1521
1522
1523
1524
1525 public String getPath() {
1526 return path;
1527 }
1528
1529
1530
1531
1532
1533 public static class CreateAndFailSilent extends ZKUtilOp {
1534 private byte [] data;
1535
1536 private CreateAndFailSilent(String path, byte [] data) {
1537 super(path);
1538 this.data = data;
1539 }
1540
1541 public byte[] getData() {
1542 return data;
1543 }
1544
1545 @Override
1546 public boolean equals(Object o) {
1547 if (this == o) return true;
1548 if (!(o instanceof CreateAndFailSilent)) return false;
1549
1550 CreateAndFailSilent op = (CreateAndFailSilent) o;
1551 return getPath().equals(op.getPath()) && Arrays.equals(data, op.data);
1552 }
1553
1554 @Override
1555 public int hashCode() {
1556 int ret = 17 + getPath().hashCode() * 31;
1557 return ret * 31 + Bytes.hashCode(data);
1558 }
1559 }
1560
1561
1562
1563
1564
1565 public static class DeleteNodeFailSilent extends ZKUtilOp {
1566 private DeleteNodeFailSilent(String path) {
1567 super(path);
1568 }
1569
1570 @Override
1571 public boolean equals(Object o) {
1572 if (this == o) return true;
1573 if (!(o instanceof DeleteNodeFailSilent)) return false;
1574
1575 return super.equals(o);
1576 }
1577
1578 @Override
1579 public int hashCode() {
1580 return getPath().hashCode();
1581 }
1582 }
1583
1584
1585
1586
1587 public static class SetData extends ZKUtilOp {
1588 private byte [] data;
1589
1590 private SetData(String path, byte [] data) {
1591 super(path);
1592 this.data = data;
1593 }
1594
1595 public byte[] getData() {
1596 return data;
1597 }
1598
1599 @Override
1600 public boolean equals(Object o) {
1601 if (this == o) return true;
1602 if (!(o instanceof SetData)) return false;
1603
1604 SetData op = (SetData) o;
1605 return getPath().equals(op.getPath()) && Arrays.equals(data, op.data);
1606 }
1607
1608 @Override
1609 public int hashCode() {
1610 int ret = getPath().hashCode();
1611 return ret * 31 + Bytes.hashCode(data);
1612 }
1613 }
1614 }
1615
1616
1617
1618
1619 private static Op toZooKeeperOp(ZooKeeperWatcher zkw, ZKUtilOp op)
1620 throws UnsupportedOperationException {
1621 if(op == null) return null;
1622
1623 if (op instanceof CreateAndFailSilent) {
1624 CreateAndFailSilent cafs = (CreateAndFailSilent)op;
1625 return Op.create(cafs.getPath(), cafs.getData(), createACL(zkw, cafs.getPath()),
1626 CreateMode.PERSISTENT);
1627 } else if (op instanceof DeleteNodeFailSilent) {
1628 DeleteNodeFailSilent dnfs = (DeleteNodeFailSilent)op;
1629 return Op.delete(dnfs.getPath(), -1);
1630 } else if (op instanceof SetData) {
1631 SetData sd = (SetData)op;
1632 return Op.setData(sd.getPath(), sd.getData(), -1);
1633 } else {
1634 throw new UnsupportedOperationException("Unexpected ZKUtilOp type: "
1635 + op.getClass().getName());
1636 }
1637 }
1638
1639
1640
1641
1642
1643
1644
1645
1646
1647
1648
1649
1650
1651
1652
1653
1654
1655
1656
1657
1658
1659
1660 public static void multiOrSequential(ZooKeeperWatcher zkw, List<ZKUtilOp> ops,
1661 boolean runSequentialOnMultiFailure) throws KeeperException {
1662 if (ops == null) return;
1663 boolean useMulti = zkw.getConfiguration().getBoolean(HConstants.ZOOKEEPER_USEMULTI, false);
1664
1665 if (useMulti) {
1666 List<Op> zkOps = new LinkedList<Op>();
1667 for (ZKUtilOp op : ops) {
1668 zkOps.add(toZooKeeperOp(zkw, op));
1669 }
1670 try {
1671 zkw.getRecoverableZooKeeper().multi(zkOps);
1672 } catch (KeeperException ke) {
1673 switch (ke.code()) {
1674 case NODEEXISTS:
1675 case NONODE:
1676 case BADVERSION:
1677 case NOAUTH:
1678
1679
1680 if (runSequentialOnMultiFailure) {
1681 LOG.info("On call to ZK.multi, received exception: " + ke.toString() + "."
1682 + " Attempting to run operations sequentially because"
1683 + " runSequentialOnMultiFailure is: " + runSequentialOnMultiFailure + ".");
1684 processSequentially(zkw, ops);
1685 break;
1686 }
1687 default:
1688 throw ke;
1689 }
1690 } catch (InterruptedException ie) {
1691 zkw.interruptedException(ie);
1692 }
1693 } else {
1694
1695 processSequentially(zkw, ops);
1696 }
1697
1698 }
1699
1700 private static void processSequentially(ZooKeeperWatcher zkw, List<ZKUtilOp> ops)
1701 throws KeeperException, NoNodeException {
1702 for (ZKUtilOp op : ops) {
1703 if (op instanceof CreateAndFailSilent) {
1704 createAndFailSilent(zkw, (CreateAndFailSilent) op);
1705 } else if (op instanceof DeleteNodeFailSilent) {
1706 deleteNodeFailSilent(zkw, (DeleteNodeFailSilent) op);
1707 } else if (op instanceof SetData) {
1708 setData(zkw, (SetData) op);
1709 } else {
1710 throw new UnsupportedOperationException("Unexpected ZKUtilOp type: "
1711 + op.getClass().getName());
1712 }
1713 }
1714 }
1715
1716
1717
1718
1719
1720
1721 public static String dump(ZooKeeperWatcher zkw) {
1722 StringBuilder sb = new StringBuilder();
1723 try {
1724 sb.append("HBase is rooted at ").append(zkw.baseZNode);
1725 sb.append("\nActive master address: ");
1726 try {
1727 sb.append(MasterAddressTracker.getMasterAddress(zkw));
1728 } catch (IOException e) {
1729 sb.append("<<FAILED LOOKUP: " + e.getMessage() + ">>");
1730 }
1731 sb.append("\nBackup master addresses:");
1732 for (String child : listChildrenNoWatch(zkw,
1733 zkw.backupMasterAddressesZNode)) {
1734 sb.append("\n ").append(child);
1735 }
1736 sb.append("\nRegion server holding hbase:meta: "
1737 + new MetaTableLocator().getMetaRegionLocation(zkw));
1738 Configuration conf = HBaseConfiguration.create();
1739 int numMetaReplicas = conf.getInt(HConstants.META_REPLICAS_NUM,
1740 HConstants.DEFAULT_META_REPLICA_NUM);
1741 for (int i = 1; i < numMetaReplicas; i++) {
1742 sb.append("\nRegion server holding hbase:meta, replicaId " + i + " "
1743 + new MetaTableLocator().getMetaRegionLocation(zkw, i));
1744 }
1745 sb.append("\nRegion servers:");
1746 for (String child : listChildrenNoWatch(zkw, zkw.rsZNode)) {
1747 sb.append("\n ").append(child);
1748 }
1749 try {
1750 getReplicationZnodesDump(zkw, sb);
1751 } catch (KeeperException ke) {
1752 LOG.warn("Couldn't get the replication znode dump", ke);
1753 }
1754 sb.append("\nQuorum Server Statistics:");
1755 String[] servers = zkw.getQuorum().split(",");
1756 for (String server : servers) {
1757 sb.append("\n ").append(server);
1758 try {
1759 String[] stat = getServerStats(server, ZKUtil.zkDumpConnectionTimeOut);
1760
1761 if (stat == null) {
1762 sb.append("[Error] invalid quorum server: " + server);
1763 break;
1764 }
1765
1766 for (String s : stat) {
1767 sb.append("\n ").append(s);
1768 }
1769 } catch (Exception e) {
1770 sb.append("\n ERROR: ").append(e.getMessage());
1771 }
1772 }
1773 } catch (KeeperException ke) {
1774 sb.append("\nFATAL ZooKeeper Exception!\n");
1775 sb.append("\n" + ke.getMessage());
1776 }
1777 return sb.toString();
1778 }
1779
1780
1781
1782
1783
1784
1785
1786 private static void getReplicationZnodesDump(ZooKeeperWatcher zkw, StringBuilder sb)
1787 throws KeeperException {
1788 String replicationZNodeName = zkw.getConfiguration().get("zookeeper.znode.replication",
1789 "replication");
1790 String replicationZnode = joinZNode(zkw.baseZNode, replicationZNodeName);
1791 if (ZKUtil.checkExists(zkw, replicationZnode) == -1) return;
1792
1793 sb.append("\n").append(replicationZnode).append(": ");
1794 List<String> children = ZKUtil.listChildrenNoWatch(zkw, replicationZnode);
1795 for (String child : children) {
1796 String znode = joinZNode(replicationZnode, child);
1797 if (child.equals(zkw.getConfiguration().get("zookeeper.znode.replication.peers", "peers"))) {
1798 appendPeersZnodes(zkw, znode, sb);
1799 } else if (child.equals(zkw.getConfiguration().
1800 get("zookeeper.znode.replication.rs", "rs"))) {
1801 appendRSZnodes(zkw, znode, sb);
1802 } else if (child.equals(zkw.getConfiguration().get(
1803 ReplicationStateZKBase.ZOOKEEPER_ZNODE_REPLICATION_HFILE_REFS_KEY,
1804 ReplicationStateZKBase.ZOOKEEPER_ZNODE_REPLICATION_HFILE_REFS_DEFAULT))) {
1805 appendHFileRefsZnodes(zkw, znode, sb);
1806 }
1807 }
1808 }
1809
1810 private static void appendHFileRefsZnodes(ZooKeeperWatcher zkw, String hfileRefsZnode,
1811 StringBuilder sb) throws KeeperException {
1812 sb.append("\n").append(hfileRefsZnode).append(": ");
1813 for (String peerIdZnode : ZKUtil.listChildrenNoWatch(zkw, hfileRefsZnode)) {
1814 String znodeToProcess = ZKUtil.joinZNode(hfileRefsZnode, peerIdZnode);
1815 sb.append("\n").append(znodeToProcess).append(": ");
1816 List<String> peerHFileRefsZnodes = ZKUtil.listChildrenNoWatch(zkw, znodeToProcess);
1817 int size = peerHFileRefsZnodes.size();
1818 for (int i = 0; i < size; i++) {
1819 sb.append(peerHFileRefsZnodes.get(i));
1820 if (i != size - 1) {
1821 sb.append(", ");
1822 }
1823 }
1824 }
1825 }
1826
1827 private static void appendRSZnodes(ZooKeeperWatcher zkw, String znode, StringBuilder sb)
1828 throws KeeperException {
1829 List<String> stack = new LinkedList<String>();
1830 stack.add(znode);
1831 do {
1832 String znodeToProcess = stack.remove(stack.size() - 1);
1833 sb.append("\n").append(znodeToProcess).append(": ");
1834 byte[] data;
1835 try {
1836 data = ZKUtil.getData(zkw, znodeToProcess);
1837 } catch (InterruptedException e) {
1838 zkw.interruptedException(e);
1839 return;
1840 }
1841 if (data != null && data.length > 0) {
1842 long position = 0;
1843 try {
1844 position = ZKUtil.parseWALPositionFrom(ZKUtil.getData(zkw, znodeToProcess));
1845 sb.append(position);
1846 } catch (DeserializationException ignored) {
1847 } catch (InterruptedException e) {
1848 zkw.interruptedException(e);
1849 return;
1850 }
1851 }
1852 for (String zNodeChild : ZKUtil.listChildrenNoWatch(zkw, znodeToProcess)) {
1853 stack.add(ZKUtil.joinZNode(znodeToProcess, zNodeChild));
1854 }
1855 } while (stack.size() > 0);
1856 }
1857
1858 private static void appendPeersZnodes(ZooKeeperWatcher zkw, String peersZnode,
1859 StringBuilder sb) throws KeeperException {
1860 int pblen = ProtobufUtil.lengthOfPBMagic();
1861 sb.append("\n").append(peersZnode).append(": ");
1862 for (String peerIdZnode : ZKUtil.listChildrenNoWatch(zkw, peersZnode)) {
1863 String znodeToProcess = ZKUtil.joinZNode(peersZnode, peerIdZnode);
1864 byte[] data;
1865 try {
1866 data = ZKUtil.getData(zkw, znodeToProcess);
1867 } catch (InterruptedException e) {
1868 zkw.interruptedException(e);
1869 return;
1870 }
1871
1872 try {
1873 ZooKeeperProtos.ReplicationPeer.Builder builder =
1874 ZooKeeperProtos.ReplicationPeer.newBuilder();
1875 ProtobufUtil.mergeFrom(builder, data, pblen, data.length - pblen);
1876 String clusterKey = builder.getClusterkey();
1877 sb.append("\n").append(znodeToProcess).append(": ").append(clusterKey);
1878
1879 appendPeerState(zkw, znodeToProcess, sb);
1880 } catch (IOException ipbe) {
1881 LOG.warn("Got Exception while parsing peer: " + znodeToProcess, ipbe);
1882 }
1883 }
1884 }
1885
1886 private static void appendPeerState(ZooKeeperWatcher zkw, String znodeToProcess,
1887 StringBuilder sb) throws KeeperException, InvalidProtocolBufferException {
1888 String peerState = zkw.getConfiguration().get("zookeeper.znode.replication.peers.state",
1889 "peer-state");
1890 int pblen = ProtobufUtil.lengthOfPBMagic();
1891 for (String child : ZKUtil.listChildrenNoWatch(zkw, znodeToProcess)) {
1892 if (!child.equals(peerState)) continue;
1893 String peerStateZnode = ZKUtil.joinZNode(znodeToProcess, child);
1894 sb.append("\n").append(peerStateZnode).append(": ");
1895 byte[] peerStateData;
1896 try {
1897 peerStateData = ZKUtil.getData(zkw, peerStateZnode);
1898 ZooKeeperProtos.ReplicationState.Builder builder =
1899 ZooKeeperProtos.ReplicationState.newBuilder();
1900 ProtobufUtil.mergeFrom(builder, peerStateData, pblen, peerStateData.length - pblen);
1901 sb.append(builder.getState().name());
1902 } catch (IOException ipbe) {
1903 LOG.warn("Got Exception while parsing peer: " + znodeToProcess, ipbe);
1904 } catch (InterruptedException e) {
1905 zkw.interruptedException(e);
1906 return;
1907 }
1908 }
1909 }
1910
1911
1912
1913
1914
1915
1916
1917
1918
1919 public static String[] getServerStats(String server, int timeout)
1920 throws IOException {
1921 String[] sp = server.split(":");
1922 if (sp == null || sp.length == 0) {
1923 return null;
1924 }
1925
1926 String host = sp[0];
1927 int port = sp.length > 1 ? Integer.parseInt(sp[1])
1928 : HConstants.DEFAULT_ZOOKEPER_CLIENT_PORT;
1929
1930 Socket socket = new Socket();
1931 InetSocketAddress sockAddr = new InetSocketAddress(host, port);
1932 socket.connect(sockAddr, timeout);
1933
1934 socket.setSoTimeout(timeout);
1935 PrintWriter out = new PrintWriter(socket.getOutputStream(), true);
1936 BufferedReader in = new BufferedReader(new InputStreamReader(
1937 socket.getInputStream()));
1938 out.println("stat");
1939 out.flush();
1940 ArrayList<String> res = new ArrayList<String>();
1941 while (true) {
1942 String line = in.readLine();
1943 if (line != null) {
1944 res.add(line);
1945 } else {
1946 break;
1947 }
1948 }
1949 socket.close();
1950 return res.toArray(new String[res.size()]);
1951 }
1952
1953 private static void logRetrievedMsg(final ZooKeeperWatcher zkw,
1954 final String znode, final byte [] data, final boolean watcherSet) {
1955 if (!LOG.isTraceEnabled()) return;
1956 LOG.trace(zkw.prefix("Retrieved " + ((data == null)? 0: data.length) +
1957 " byte(s) of data from znode " + znode +
1958 (watcherSet? " and set watcher; ": "; data=") +
1959 (data == null? "null": data.length == 0? "empty": (
1960 znode.startsWith(zkw.assignmentZNode)?
1961 ZKAssign.toString(data):
1962 znode.startsWith(ZooKeeperWatcher.META_ZNODE_PREFIX)?
1963 getServerNameOrEmptyString(data):
1964 znode.startsWith(zkw.backupMasterAddressesZNode)?
1965 getServerNameOrEmptyString(data):
1966 StringUtils.abbreviate(Bytes.toStringBinary(data), 32)))));
1967 }
1968
1969 private static String getServerNameOrEmptyString(final byte [] data) {
1970 try {
1971 return ServerName.parseFrom(data).toString();
1972 } catch (DeserializationException e) {
1973 return "";
1974 }
1975 }
1976
1977
1978
1979
1980
1981 public static void waitForBaseZNode(Configuration conf) throws IOException {
1982 LOG.info("Waiting until the base znode is available");
1983 String parentZNode = conf.get(HConstants.ZOOKEEPER_ZNODE_PARENT,
1984 HConstants.DEFAULT_ZOOKEEPER_ZNODE_PARENT);
1985 ZooKeeper zk = new ZooKeeper(ZKConfig.getZKQuorumServersString(conf),
1986 conf.getInt(HConstants.ZK_SESSION_TIMEOUT,
1987 HConstants.DEFAULT_ZK_SESSION_TIMEOUT), EmptyWatcher.instance);
1988
1989 final int maxTimeMs = 10000;
1990 final int maxNumAttempts = maxTimeMs / HConstants.SOCKET_RETRY_WAIT_MS;
1991
1992 KeeperException keeperEx = null;
1993 try {
1994 try {
1995 for (int attempt = 0; attempt < maxNumAttempts; ++attempt) {
1996 try {
1997 if (zk.exists(parentZNode, false) != null) {
1998 LOG.info("Parent znode exists: " + parentZNode);
1999 keeperEx = null;
2000 break;
2001 }
2002 } catch (KeeperException e) {
2003 keeperEx = e;
2004 }
2005 Threads.sleepWithoutInterrupt(HConstants.SOCKET_RETRY_WAIT_MS);
2006 }
2007 } finally {
2008 zk.close();
2009 }
2010 } catch (InterruptedException ex) {
2011 Thread.currentThread().interrupt();
2012 }
2013
2014 if (keeperEx != null) {
2015 throw new IOException(keeperEx);
2016 }
2017 }
2018
2019
2020 public static byte[] blockUntilAvailable(
2021 final ZooKeeperWatcher zkw, final String znode, final long timeout)
2022 throws InterruptedException {
2023 if (timeout < 0) throw new IllegalArgumentException();
2024 if (zkw == null) throw new IllegalArgumentException();
2025 if (znode == null) throw new IllegalArgumentException();
2026
2027 byte[] data = null;
2028 boolean finished = false;
2029 final long endTime = System.currentTimeMillis() + timeout;
2030 while (!finished) {
2031 try {
2032 data = ZKUtil.getData(zkw, znode);
2033 } catch(KeeperException e) {
2034 if (e instanceof KeeperException.SessionExpiredException
2035 || e instanceof KeeperException.AuthFailedException) {
2036
2037 throw new InterruptedException("interrupted due to " + e);
2038 }
2039 LOG.warn("Unexpected exception handling blockUntilAvailable", e);
2040 }
2041
2042 if (data == null && (System.currentTimeMillis() +
2043 HConstants.SOCKET_RETRY_WAIT_MS < endTime)) {
2044 Thread.sleep(HConstants.SOCKET_RETRY_WAIT_MS);
2045 } else {
2046 finished = true;
2047 }
2048 }
2049
2050 return data;
2051 }
2052
2053
2054
2055
2056
2057
2058
2059
2060 public static KeeperException convert(final DeserializationException e) {
2061 KeeperException ke = new KeeperException.DataInconsistencyException();
2062 ke.initCause(e);
2063 return ke;
2064 }
2065
2066
2067
2068
2069
2070
2071 public static void logZKTree(ZooKeeperWatcher zkw, String root) {
2072 if (!LOG.isDebugEnabled()) return;
2073 LOG.debug("Current zk system:");
2074 String prefix = "|-";
2075 LOG.debug(prefix + root);
2076 try {
2077 logZKTree(zkw, root, prefix);
2078 } catch (KeeperException e) {
2079 throw new RuntimeException(e);
2080 }
2081 }
2082
2083
2084
2085
2086
2087
2088 protected static void logZKTree(ZooKeeperWatcher zkw, String root, String prefix)
2089 throws KeeperException {
2090 List<String> children = ZKUtil.listChildrenNoWatch(zkw, root);
2091 if (children == null) return;
2092 for (String child : children) {
2093 LOG.debug(prefix + child);
2094 String node = ZKUtil.joinZNode(root.equals("/") ? "" : root, child);
2095 logZKTree(zkw, node, prefix + "---");
2096 }
2097 }
2098
2099
2100
2101
2102
2103
2104 public static byte[] positionToByteArray(final long position) {
2105 byte[] bytes = ZooKeeperProtos.ReplicationHLogPosition.newBuilder().setPosition(position)
2106 .build().toByteArray();
2107 return ProtobufUtil.prependPBMagic(bytes);
2108 }
2109
2110
2111
2112
2113
2114
2115 public static long parseWALPositionFrom(final byte[] bytes) throws DeserializationException {
2116 if (bytes == null) {
2117 throw new DeserializationException("Unable to parse null WAL position.");
2118 }
2119 if (ProtobufUtil.isPBMagicPrefix(bytes)) {
2120 int pblen = ProtobufUtil.lengthOfPBMagic();
2121 ZooKeeperProtos.ReplicationHLogPosition.Builder builder =
2122 ZooKeeperProtos.ReplicationHLogPosition.newBuilder();
2123 ZooKeeperProtos.ReplicationHLogPosition position;
2124 try {
2125 ProtobufUtil.mergeFrom(builder, bytes, pblen, bytes.length - pblen);
2126 position = builder.build();
2127 } catch (IOException e) {
2128 throw new DeserializationException(e);
2129 }
2130 return position.getPosition();
2131 } else {
2132 if (bytes.length > 0) {
2133 return Bytes.toLong(bytes);
2134 }
2135 return 0;
2136 }
2137 }
2138
2139
2140
2141
2142
2143
2144
2145
2146 public static byte[] regionSequenceIdsToByteArray(final Long regionLastFlushedSequenceId,
2147 final Map<byte[], Long> storeSequenceIds) {
2148 ClusterStatusProtos.RegionStoreSequenceIds.Builder regionSequenceIdsBuilder =
2149 ClusterStatusProtos.RegionStoreSequenceIds.newBuilder();
2150 ClusterStatusProtos.StoreSequenceId.Builder storeSequenceIdBuilder =
2151 ClusterStatusProtos.StoreSequenceId.newBuilder();
2152 if (storeSequenceIds != null) {
2153 for (Map.Entry<byte[], Long> e : storeSequenceIds.entrySet()){
2154 byte[] columnFamilyName = e.getKey();
2155 Long curSeqId = e.getValue();
2156 storeSequenceIdBuilder.setFamilyName(ByteStringer.wrap(columnFamilyName));
2157 storeSequenceIdBuilder.setSequenceId(curSeqId);
2158 regionSequenceIdsBuilder.addStoreSequenceId(storeSequenceIdBuilder.build());
2159 storeSequenceIdBuilder.clear();
2160 }
2161 }
2162 regionSequenceIdsBuilder.setLastFlushedSequenceId(regionLastFlushedSequenceId);
2163 byte[] result = regionSequenceIdsBuilder.build().toByteArray();
2164 return ProtobufUtil.prependPBMagic(result);
2165 }
2166
2167
2168
2169
2170
2171
2172 public static RegionStoreSequenceIds parseRegionStoreSequenceIds(final byte[] bytes)
2173 throws DeserializationException {
2174 if (bytes == null || !ProtobufUtil.isPBMagicPrefix(bytes)) {
2175 throw new DeserializationException("Unable to parse RegionStoreSequenceIds.");
2176 }
2177 RegionStoreSequenceIds.Builder regionSequenceIdsBuilder =
2178 ClusterStatusProtos.RegionStoreSequenceIds.newBuilder();
2179 int pblen = ProtobufUtil.lengthOfPBMagic();
2180 RegionStoreSequenceIds storeIds = null;
2181 try {
2182 ProtobufUtil.mergeFrom(regionSequenceIdsBuilder, bytes, pblen, bytes.length - pblen);
2183 storeIds = regionSequenceIdsBuilder.build();
2184 } catch (IOException e) {
2185 throw new DeserializationException(e);
2186 }
2187 return storeIds;
2188 }
2189 }