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 static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertNotEquals;
022import static org.junit.Assert.assertTrue;
023
024import java.util.ArrayList;
025import java.util.Arrays;
026import java.util.Collections;
027import java.util.LinkedList;
028import java.util.List;
029
030import org.apache.hadoop.conf.Configuration;
031import org.apache.hadoop.hbase.Abortable;
032import org.apache.hadoop.hbase.HBaseClassTestRule;
033import org.apache.hadoop.hbase.HBaseZKTestingUtility;
034import org.apache.hadoop.hbase.testclassification.MediumTests;
035import org.apache.hadoop.hbase.testclassification.ZKTests;
036import org.apache.hadoop.hbase.util.Bytes;
037import org.apache.hadoop.hbase.zookeeper.ZKUtil.ZKUtilOp;
038import org.apache.zookeeper.CreateMode;
039import org.apache.zookeeper.KeeperException;
040import org.apache.zookeeper.Op;
041import org.apache.zookeeper.ZooDefs.Ids;
042import org.junit.AfterClass;
043import org.junit.BeforeClass;
044import org.junit.ClassRule;
045import org.junit.Test;
046import org.junit.experimental.categories.Category;
047import org.slf4j.Logger;
048import org.slf4j.LoggerFactory;
049
050/**
051 * Test ZooKeeper multi-update functionality.
052 */
053@Category({ ZKTests.class, MediumTests.class })
054public class TestZKMulti {
055  @ClassRule
056  public static final HBaseClassTestRule CLASS_RULE =
057      HBaseClassTestRule.forClass(TestZKMulti.class);
058
059  private static final Logger LOG = LoggerFactory.getLogger(TestZKMulti.class);
060  private final static HBaseZKTestingUtility TEST_UTIL = new HBaseZKTestingUtility();
061  private static ZKWatcher zkw = null;
062
063  private static class ZKMultiAbortable implements Abortable {
064    @Override
065    public void abort(String why, Throwable e) {
066      LOG.info(why, e);
067    }
068
069    @Override
070    public boolean isAborted() {
071      return false;
072    }
073  }
074
075  @BeforeClass
076  public static void setUpBeforeClass() throws Exception {
077    TEST_UTIL.startMiniZKCluster();
078    Configuration conf = TEST_UTIL.getConfiguration();
079    Abortable abortable = new ZKMultiAbortable();
080    zkw = new ZKWatcher(conf,
081      "TestZKMulti", abortable, true);
082  }
083
084  @AfterClass
085  public static void tearDownAfterClass() throws Exception {
086    TEST_UTIL.shutdownMiniZKCluster();
087  }
088
089  @Test
090  public void testSimpleMulti() throws Exception {
091    // null multi
092    ZKUtil.multiOrSequential(zkw, null, false);
093
094    // empty multi
095    ZKUtil.multiOrSequential(zkw, new LinkedList<>(), false);
096
097    // single create
098    String path = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testSimpleMulti");
099    LinkedList<ZKUtilOp> singleCreate = new LinkedList<>();
100    singleCreate.add(ZKUtilOp.createAndFailSilent(path, new byte[0]));
101    ZKUtil.multiOrSequential(zkw, singleCreate, false);
102    assertTrue(ZKUtil.checkExists(zkw, path) != -1);
103
104    // single setdata
105    LinkedList<ZKUtilOp> singleSetData = new LinkedList<>();
106    byte [] data = Bytes.toBytes("foobar");
107    singleSetData.add(ZKUtilOp.setData(path, data));
108    ZKUtil.multiOrSequential(zkw, singleSetData, false);
109    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path), data));
110
111    // single delete
112    LinkedList<ZKUtilOp> singleDelete = new LinkedList<>();
113    singleDelete.add(ZKUtilOp.deleteNodeFailSilent(path));
114    ZKUtil.multiOrSequential(zkw, singleDelete, false);
115    assertEquals(-1, ZKUtil.checkExists(zkw, path));
116  }
117
118  @Test
119  public void testComplexMulti() throws Exception {
120    String path1 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testComplexMulti1");
121    String path2 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testComplexMulti2");
122    String path3 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testComplexMulti3");
123    String path4 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testComplexMulti4");
124    String path5 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testComplexMulti5");
125    String path6 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testComplexMulti6");
126    // create 4 nodes that we'll setData on or delete later
127    LinkedList<ZKUtilOp> create4Nodes = new LinkedList<>();
128    create4Nodes.add(ZKUtilOp.createAndFailSilent(path1, Bytes.toBytes(path1)));
129    create4Nodes.add(ZKUtilOp.createAndFailSilent(path2, Bytes.toBytes(path2)));
130    create4Nodes.add(ZKUtilOp.createAndFailSilent(path3, Bytes.toBytes(path3)));
131    create4Nodes.add(ZKUtilOp.createAndFailSilent(path4, Bytes.toBytes(path4)));
132    ZKUtil.multiOrSequential(zkw, create4Nodes, false);
133    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path1), Bytes.toBytes(path1)));
134    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path2), Bytes.toBytes(path2)));
135    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path3), Bytes.toBytes(path3)));
136    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path4), Bytes.toBytes(path4)));
137
138    // do multiple of each operation (setData, delete, create)
139    LinkedList<ZKUtilOp> ops = new LinkedList<>();
140    // setData
141    ops.add(ZKUtilOp.setData(path1, Bytes.add(Bytes.toBytes(path1), Bytes.toBytes(path1))));
142    ops.add(ZKUtilOp.setData(path2, Bytes.add(Bytes.toBytes(path2), Bytes.toBytes(path2))));
143    // delete
144    ops.add(ZKUtilOp.deleteNodeFailSilent(path3));
145    ops.add(ZKUtilOp.deleteNodeFailSilent(path4));
146    // create
147    ops.add(ZKUtilOp.createAndFailSilent(path5, Bytes.toBytes(path5)));
148    ops.add(ZKUtilOp.createAndFailSilent(path6, Bytes.toBytes(path6)));
149    ZKUtil.multiOrSequential(zkw, ops, false);
150    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path1),
151      Bytes.add(Bytes.toBytes(path1), Bytes.toBytes(path1))));
152    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path2),
153      Bytes.add(Bytes.toBytes(path2), Bytes.toBytes(path2))));
154    assertEquals(-1, ZKUtil.checkExists(zkw, path3));
155    assertEquals(-1, ZKUtil.checkExists(zkw, path4));
156    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path5), Bytes.toBytes(path5)));
157    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path6), Bytes.toBytes(path6)));
158  }
159
160  @Test
161  public void testSingleFailure() throws Exception {
162    // try to delete a node that doesn't exist
163    boolean caughtNoNode = false;
164    String path = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testSingleFailureZ");
165    LinkedList<ZKUtilOp> ops = new LinkedList<>();
166    ops.add(ZKUtilOp.deleteNodeFailSilent(path));
167    try {
168      ZKUtil.multiOrSequential(zkw, ops, false);
169    } catch (KeeperException.NoNodeException nne) {
170      caughtNoNode = true;
171    }
172    assertTrue(caughtNoNode);
173
174    // try to setData on a node that doesn't exist
175    caughtNoNode = false;
176    ops = new LinkedList<>();
177    ops.add(ZKUtilOp.setData(path, Bytes.toBytes(path)));
178    try {
179      ZKUtil.multiOrSequential(zkw, ops, false);
180    } catch (KeeperException.NoNodeException nne) {
181      caughtNoNode = true;
182    }
183    assertTrue(caughtNoNode);
184
185    // try to create on a node that already exists
186    boolean caughtNodeExists = false;
187    ops = new LinkedList<>();
188    ops.add(ZKUtilOp.createAndFailSilent(path, Bytes.toBytes(path)));
189    ZKUtil.multiOrSequential(zkw, ops, false);
190    try {
191      ZKUtil.multiOrSequential(zkw, ops, false);
192    } catch (KeeperException.NodeExistsException nee) {
193      caughtNodeExists = true;
194    }
195    assertTrue(caughtNodeExists);
196  }
197
198  @Test
199  public void testSingleFailureInMulti() throws Exception {
200    // try a multi where all but one operation succeeds
201    String pathA = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testSingleFailureInMultiA");
202    String pathB = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testSingleFailureInMultiB");
203    String pathC = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testSingleFailureInMultiC");
204    LinkedList<ZKUtilOp> ops = new LinkedList<>();
205    ops.add(ZKUtilOp.createAndFailSilent(pathA, Bytes.toBytes(pathA)));
206    ops.add(ZKUtilOp.createAndFailSilent(pathB, Bytes.toBytes(pathB)));
207    ops.add(ZKUtilOp.deleteNodeFailSilent(pathC));
208    boolean caughtNoNode = false;
209    try {
210      ZKUtil.multiOrSequential(zkw, ops, false);
211    } catch (KeeperException.NoNodeException nne) {
212      caughtNoNode = true;
213    }
214    assertTrue(caughtNoNode);
215    // assert that none of the operations succeeded
216    assertEquals(-1, ZKUtil.checkExists(zkw, pathA));
217    assertEquals(-1, ZKUtil.checkExists(zkw, pathB));
218    assertEquals(-1, ZKUtil.checkExists(zkw, pathC));
219  }
220
221  @Test
222  public void testMultiFailure() throws Exception {
223    String pathX = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testMultiFailureX");
224    String pathY = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testMultiFailureY");
225    String pathZ = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testMultiFailureZ");
226    // create X that we will use to fail create later
227    LinkedList<ZKUtilOp> ops = new LinkedList<>();
228    ops.add(ZKUtilOp.createAndFailSilent(pathX, Bytes.toBytes(pathX)));
229    ZKUtil.multiOrSequential(zkw, ops, false);
230
231    // fail one of each create ,setData, delete
232    String pathV = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testMultiFailureV");
233    String pathW = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "testMultiFailureW");
234    ops = new LinkedList<>();
235    ops.add(ZKUtilOp.createAndFailSilent(pathX, Bytes.toBytes(pathX))); // fail  -- already exists
236    ops.add(ZKUtilOp.setData(pathY, Bytes.toBytes(pathY))); // fail -- doesn't exist
237    ops.add(ZKUtilOp.deleteNodeFailSilent(pathZ)); // fail -- doesn't exist
238    ops.add(ZKUtilOp.createAndFailSilent(pathX, Bytes.toBytes(pathV))); // pass
239    ops.add(ZKUtilOp.createAndFailSilent(pathX, Bytes.toBytes(pathW))); // pass
240    boolean caughtNodeExists = false;
241    try {
242      ZKUtil.multiOrSequential(zkw, ops, false);
243    } catch (KeeperException.NodeExistsException nee) {
244      // check first operation that fails throws exception
245      caughtNodeExists = true;
246    }
247    assertTrue(caughtNodeExists);
248    // check that no modifications were made
249    assertNotEquals(-1, ZKUtil.checkExists(zkw, pathX));
250    assertEquals(-1, ZKUtil.checkExists(zkw, pathY));
251    assertEquals(-1, ZKUtil.checkExists(zkw, pathZ));
252    assertEquals(-1, ZKUtil.checkExists(zkw, pathW));
253    assertEquals(-1, ZKUtil.checkExists(zkw, pathV));
254
255    // test that with multiple failures, throws an exception corresponding to first failure in list
256    ops = new LinkedList<>();
257    ops.add(ZKUtilOp.setData(pathY, Bytes.toBytes(pathY))); // fail -- doesn't exist
258    ops.add(ZKUtilOp.createAndFailSilent(pathX, Bytes.toBytes(pathX))); // fail -- exists
259    boolean caughtNoNode = false;
260    try {
261      ZKUtil.multiOrSequential(zkw, ops, false);
262    } catch (KeeperException.NoNodeException nne) {
263      // check first operation that fails throws exception
264      caughtNoNode = true;
265    }
266    assertTrue(caughtNoNode);
267    // check that no modifications were made
268    assertNotEquals(-1, ZKUtil.checkExists(zkw, pathX));
269    assertEquals(-1, ZKUtil.checkExists(zkw, pathY));
270    assertEquals(-1, ZKUtil.checkExists(zkw, pathZ));
271    assertEquals(-1, ZKUtil.checkExists(zkw, pathW));
272    assertEquals(-1, ZKUtil.checkExists(zkw, pathV));
273  }
274
275  @Test
276  public void testRunSequentialOnMultiFailure() throws Exception {
277    String path1 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "runSequential1");
278    String path2 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "runSequential2");
279    String path3 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "runSequential3");
280    String path4 = ZNodePaths.joinZNode(zkw.getZNodePaths().baseZNode, "runSequential4");
281
282    // create some nodes that we will use later
283    LinkedList<ZKUtilOp> ops = new LinkedList<>();
284    ops.add(ZKUtilOp.createAndFailSilent(path1, Bytes.toBytes(path1)));
285    ops.add(ZKUtilOp.createAndFailSilent(path2, Bytes.toBytes(path2)));
286    ZKUtil.multiOrSequential(zkw, ops, false);
287
288    // test that, even with operations that fail, the ones that would pass will pass
289    // with runSequentialOnMultiFailure
290    ops = new LinkedList<>();
291    ops.add(ZKUtilOp.setData(path1, Bytes.add(Bytes.toBytes(path1), Bytes.toBytes(path1)))); // pass
292    ops.add(ZKUtilOp.deleteNodeFailSilent(path2)); // pass
293    ops.add(ZKUtilOp.deleteNodeFailSilent(path3)); // fail -- node doesn't exist
294    ops.add(ZKUtilOp.createAndFailSilent(path4,
295      Bytes.add(Bytes.toBytes(path4), Bytes.toBytes(path4)))); // pass
296    ZKUtil.multiOrSequential(zkw, ops, true);
297    assertTrue(Bytes.equals(ZKUtil.getData(zkw, path1),
298      Bytes.add(Bytes.toBytes(path1), Bytes.toBytes(path1))));
299    assertEquals(-1, ZKUtil.checkExists(zkw, path2));
300    assertEquals(-1, ZKUtil.checkExists(zkw, path3));
301    assertNotEquals(-1, ZKUtil.checkExists(zkw, path4));
302  }
303
304  /**
305   * Verifies that for the given root node, it should delete all the child nodes
306   * recursively using multi-update api.
307   */
308  @Test
309  public void testdeleteChildrenRecursivelyMulti() throws Exception {
310    String parentZNode = "/testRootMulti";
311    createZNodeTree(parentZNode);
312
313    ZKUtil.deleteChildrenRecursivelyMultiOrSequential(zkw, true, parentZNode);
314
315    assertTrue("Wrongly deleted parent znode!",
316        ZKUtil.checkExists(zkw, parentZNode) > -1);
317    List<String> children = zkw.getRecoverableZooKeeper().getChildren(
318        parentZNode, false);
319    assertEquals("Failed to delete child znodes!", 0, children.size());
320  }
321
322  /**
323   * Verifies that for the given root node, it should delete all the nodes recursively using
324   * multi-update api.
325   */
326  @Test
327  public void testDeleteNodeRecursivelyMulti() throws Exception {
328    String parentZNode = "/testdeleteNodeRecursivelyMulti";
329    createZNodeTree(parentZNode);
330
331    ZKUtil.deleteNodeRecursively(zkw, parentZNode);
332    assertEquals("Parent znode should be deleted.", -1, ZKUtil.checkExists(zkw, parentZNode));
333  }
334
335  @Test
336  public void testDeleteNodeRecursivelyMultiOrSequential() throws Exception {
337    String parentZNode1 = "/testdeleteNode1";
338    String parentZNode2 = "/testdeleteNode2";
339    String parentZNode3 = "/testdeleteNode3";
340    createZNodeTree(parentZNode1);
341    createZNodeTree(parentZNode2);
342    createZNodeTree(parentZNode3);
343
344    ZKUtil.deleteNodeRecursivelyMultiOrSequential(zkw, false, parentZNode1, parentZNode2,
345      parentZNode3);
346    assertEquals("Parent znode 1 should be deleted.", -1, ZKUtil.checkExists(zkw, parentZNode1));
347    assertEquals("Parent znode 2 should be deleted.", -1, ZKUtil.checkExists(zkw, parentZNode2));
348    assertEquals("Parent znode 3 should be deleted.", -1, ZKUtil.checkExists(zkw, parentZNode3));
349  }
350
351  @Test
352  public void testDeleteChildrenRecursivelyMultiOrSequential() throws Exception {
353    String parentZNode1 = "/testdeleteChildren1";
354    String parentZNode2 = "/testdeleteChildren2";
355    String parentZNode3 = "/testdeleteChildren3";
356    createZNodeTree(parentZNode1);
357    createZNodeTree(parentZNode2);
358    createZNodeTree(parentZNode3);
359
360    ZKUtil.deleteChildrenRecursivelyMultiOrSequential(zkw, true, parentZNode1, parentZNode2,
361      parentZNode3);
362
363    assertTrue("Wrongly deleted parent znode 1!", ZKUtil.checkExists(zkw, parentZNode1) > -1);
364    List<String> children = zkw.getRecoverableZooKeeper().getChildren(parentZNode1, false);
365    assertEquals("Failed to delete child znodes of parent znode 1!", 0, children.size());
366
367    assertTrue("Wrongly deleted parent znode 2!", ZKUtil.checkExists(zkw, parentZNode2) > -1);
368    children = zkw.getRecoverableZooKeeper().getChildren(parentZNode2, false);
369    assertEquals("Failed to delete child znodes of parent znode 1!", 0, children.size());
370
371    assertTrue("Wrongly deleted parent znode 3!", ZKUtil.checkExists(zkw, parentZNode3) > -1);
372    children = zkw.getRecoverableZooKeeper().getChildren(parentZNode3, false);
373    assertEquals("Failed to delete child znodes of parent znode 1!", 0, children.size());
374  }
375
376  @Test
377  public void testBatchedDeletesOfWideZNodes() throws Exception {
378    // Batch every 50bytes
379    final int batchSize = 50;
380    Configuration localConf = new Configuration(TEST_UTIL.getConfiguration());
381    localConf.setInt("zookeeper.multi.max.size", batchSize);
382    try (ZKWatcher customZkw = new ZKWatcher(localConf,
383      "TestZKMulti_Custom", new ZKMultiAbortable(), true)) {
384
385      // With a parent znode like this, we'll get batches of 2-3 elements
386      final String parent1 = "/batchedDeletes1";
387      final String parent2 = "/batchedDeletes2";
388      final byte[] EMPTY_BYTES = new byte[0];
389
390      // Write one node
391      List<Op> ops = new ArrayList<>();
392      ops.add(Op.create(parent1, EMPTY_BYTES, Ids.OPEN_ACL_UNSAFE, CreateMode.PERSISTENT));
393      for (int i = 0; i < batchSize * 2; i++) {
394        ops.add(Op.create(
395            parent1 + "/" + i, EMPTY_BYTES, Ids.OPEN_ACL_UNSAFE, CreateMode.PERSISTENT));
396      }
397      customZkw.getRecoverableZooKeeper().multi(ops);
398
399      // Write into a second node
400      ops.clear();
401      ops.add(Op.create(parent2, EMPTY_BYTES, Ids.OPEN_ACL_UNSAFE, CreateMode.PERSISTENT));
402      for (int i = 0; i < batchSize * 4; i++) {
403        ops.add(Op.create(
404            parent2 + "/" + i, EMPTY_BYTES, Ids.OPEN_ACL_UNSAFE, CreateMode.PERSISTENT));
405      }
406      customZkw.getRecoverableZooKeeper().multi(ops);
407
408      // These should return successfully
409      ZKUtil.deleteChildrenRecursively(customZkw, parent1);
410      ZKUtil.deleteChildrenRecursively(customZkw, parent2);
411    }
412  }
413
414  @Test
415  public void testListPartitioning() {
416    // 10 Bytes
417    ZKUtilOp tenByteOp = ZKUtilOp.deleteNodeFailSilent("/123456789");
418
419    // Simple, single element case
420    assertEquals(Collections.singletonList(Collections.singletonList(tenByteOp)),
421        ZKUtil.partitionOps(Collections.singletonList(tenByteOp), 15));
422
423    // Simple case where we exceed the limit, but must make the list
424    assertEquals(Collections.singletonList(Collections.singletonList(tenByteOp)),
425        ZKUtil.partitionOps(Collections.singletonList(tenByteOp), 5));
426
427    // Each gets its own bucket
428    assertEquals(
429        Arrays.asList(Collections.singletonList(tenByteOp), Collections.singletonList(tenByteOp),
430            Collections.singletonList(tenByteOp)),
431        ZKUtil.partitionOps(Arrays.asList(tenByteOp, tenByteOp, tenByteOp), 15));
432
433    // Test internal boundary
434    assertEquals(
435        Arrays.asList(Arrays.asList(tenByteOp,tenByteOp), Collections.singletonList(tenByteOp)),
436        ZKUtil.partitionOps(Arrays.asList(tenByteOp, tenByteOp, tenByteOp), 20));
437
438    // Plenty of space for one partition
439    assertEquals(
440        Collections.singletonList(Arrays.asList(tenByteOp, tenByteOp, tenByteOp)),
441        ZKUtil.partitionOps(Arrays.asList(tenByteOp, tenByteOp, tenByteOp), 50));
442  }
443
444  private void createZNodeTree(String rootZNode) throws KeeperException,
445      InterruptedException {
446    List<Op> opList = new ArrayList<>();
447    opList.add(Op.create(rootZNode, new byte[0], Ids.OPEN_ACL_UNSAFE,
448        CreateMode.PERSISTENT));
449    int level = 0;
450    String parentZNode = rootZNode;
451    while (level < 10) {
452      // define parent node
453      parentZNode = parentZNode + "/" + level;
454      opList.add(Op.create(parentZNode, new byte[0], Ids.OPEN_ACL_UNSAFE,
455          CreateMode.PERSISTENT));
456      int elements = 0;
457      // add elements to the parent node
458      while (elements < level) {
459        opList.add(Op.create(parentZNode + "/" + elements, new byte[0],
460            Ids.OPEN_ACL_UNSAFE, CreateMode.PERSISTENT));
461        elements++;
462      }
463      level++;
464    }
465    zkw.getRecoverableZooKeeper().multi(opList);
466  }
467}