View Javadoc

1   /*
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  package org.apache.hadoop.hbase.replication;
19  
20  import static org.junit.Assert.assertEquals;
21  
22  import java.util.ArrayList;
23  import java.util.List;
24  
25  import org.apache.commons.logging.Log;
26  import org.apache.commons.logging.LogFactory;
27  import org.apache.hadoop.hbase.*;
28  import org.apache.hadoop.hbase.client.Delete;
29  import org.apache.hadoop.hbase.client.HBaseAdmin;
30  import org.apache.hadoop.hbase.client.HTable;
31  import org.apache.hadoop.hbase.client.Put;
32  import org.apache.hadoop.hbase.client.replication.ReplicationAdmin;
33  import org.apache.hadoop.hbase.util.Bytes;
34  import org.apache.hadoop.hbase.replication.regionserver.ReplicationSyncUp;
35  import org.junit.Before;
36  import org.junit.Test;
37  import org.junit.experimental.categories.Category;
38  
39  @Category(LargeTests.class)
40  public class TestReplicationSyncUpTool extends TestReplicationBase {
41  
42    private static final Log LOG = LogFactory.getLog(TestReplicationSyncUpTool.class);
43  
44    private static final byte[] t1_su = Bytes.toBytes("t1_syncup");
45    private static final byte[] t2_su = Bytes.toBytes("t2_syncup");
46  
47    private static final byte[] famName = Bytes.toBytes("cf1");
48    private static final byte[] qualName = Bytes.toBytes("q1");
49  
50    private static final byte[] noRepfamName = Bytes.toBytes("norep");
51  
52    private HTableDescriptor t1_syncupSource, t1_syncupTarget;
53    private HTableDescriptor t2_syncupSource, t2_syncupTarget;
54  
55    private HTable ht1Source, ht2Source, ht1TargetAtPeer1, ht2TargetAtPeer1;
56  
57    @Before
58    public void setUp() throws Exception {
59  
60      HColumnDescriptor fam;
61  
62      t1_syncupSource = new HTableDescriptor(TableName.valueOf(t1_su));
63      fam = new HColumnDescriptor(famName);
64      fam.setScope(HConstants.REPLICATION_SCOPE_GLOBAL);
65      t1_syncupSource.addFamily(fam);
66      fam = new HColumnDescriptor(noRepfamName);
67      t1_syncupSource.addFamily(fam);
68  
69      t1_syncupTarget = new HTableDescriptor(TableName.valueOf(t1_su));
70      fam = new HColumnDescriptor(famName);
71      t1_syncupTarget.addFamily(fam);
72      fam = new HColumnDescriptor(noRepfamName);
73      t1_syncupTarget.addFamily(fam);
74  
75      t2_syncupSource = new HTableDescriptor(TableName.valueOf(t2_su));
76      fam = new HColumnDescriptor(famName);
77      fam.setScope(HConstants.REPLICATION_SCOPE_GLOBAL);
78      t2_syncupSource.addFamily(fam);
79      fam = new HColumnDescriptor(noRepfamName);
80      t2_syncupSource.addFamily(fam);
81  
82      t2_syncupTarget = new HTableDescriptor(TableName.valueOf(t2_su));
83      fam = new HColumnDescriptor(famName);
84      t2_syncupTarget.addFamily(fam);
85      fam = new HColumnDescriptor(noRepfamName);
86      t2_syncupTarget.addFamily(fam);
87  
88    }
89  
90    /**
91     * Add a row to a table in each cluster, check it's replicated, delete it,
92     * check's gone Also check the puts and deletes are not replicated back to
93     * the originating cluster.
94     */
95    @Test(timeout = 300000)
96    public void testSyncUpTool() throws Exception {
97  
98      /**
99       * Set up Replication: on Master and one Slave
100      * Table: t1_syncup and t2_syncup
101      * columnfamily:
102      *    'cf1'  : replicated
103      *    'norep': not replicated
104      */
105     setupReplication();
106 
107     /**
108      * at Master:
109      * t1_syncup: put 100 rows into cf1, and 1 rows into norep
110      * t2_syncup: put 200 rows into cf1, and 1 rows into norep
111      *
112      * verify correctly replicated to slave
113      */
114     putAndReplicateRows();
115 
116     /**
117      * Verify delete works
118      *
119      * step 1: stop hbase on Slave
120      *
121      * step 2: at Master:
122      *  t1_syncup: delete 50 rows  from cf1
123      *  t2_syncup: delete 100 rows from cf1
124      *  no change on 'norep'
125      *
126      * step 3: stop hbase on master, restart hbase on Slave
127      *
128      * step 4: verify Slave still have the rows before delete
129      *      t1_syncup: 100 rows from cf1
130      *      t2_syncup: 200 rows from cf1
131      *
132      * step 5: run syncup tool on Master
133      *
134      * step 6: verify that delete show up on Slave
135      *      t1_syncup: 50 rows from cf1
136      *      t2_syncup: 100 rows from cf1
137      *
138      * verify correctly replicated to Slave
139      */
140     mimicSyncUpAfterDelete();
141 
142     /**
143      * Verify put works
144      *
145      * step 1: stop hbase on Slave
146      *
147      * step 2: at Master:
148      *  t1_syncup: put 100 rows  from cf1
149      *  t2_syncup: put 200 rows  from cf1
150      *  and put another row on 'norep'
151      *  ATTN: put to 'cf1' will overwrite existing rows, so end count will
152      *        be 100 and 200 respectively
153      *      put to 'norep' will add a new row.
154      *
155      * step 3: stop hbase on master, restart hbase on Slave
156      *
157      * step 4: verify Slave still has the rows before put
158      *      t1_syncup: 50 rows from cf1
159      *      t2_syncup: 100 rows from cf1
160      *
161      * step 5: run syncup tool on Master
162      *
163      * step 6: verify that put show up on Slave
164      *         and 'norep' does not
165      *      t1_syncup: 100 rows from cf1
166      *      t2_syncup: 200 rows from cf1
167      *
168      * verify correctly replicated to Slave
169      */
170     mimicSyncUpAfterPut();
171 
172   }
173 
174   private void setupReplication() throws Exception {
175     ReplicationAdmin admin1 = new ReplicationAdmin(conf1);
176     ReplicationAdmin admin2 = new ReplicationAdmin(conf2);
177 
178     HBaseAdmin ha = new HBaseAdmin(conf1);
179     ha.createTable(t1_syncupSource);
180     ha.createTable(t2_syncupSource);
181     ha.close();
182 
183     ha = new HBaseAdmin(conf2);
184     ha.createTable(t1_syncupTarget);
185     ha.createTable(t2_syncupTarget);
186     ha.close();
187 
188     // Get HTable from Master
189     ht1Source = new HTable(conf1, t1_su);
190     ht1Source.setWriteBufferSize(1024);
191     ht2Source = new HTable(conf1, t2_su);
192     ht1Source.setWriteBufferSize(1024);
193 
194     // Get HTable from Peer1
195     ht1TargetAtPeer1 = new HTable(conf2, t1_su);
196     ht1TargetAtPeer1.setWriteBufferSize(1024);
197     ht2TargetAtPeer1 = new HTable(conf2, t2_su);
198     ht2TargetAtPeer1.setWriteBufferSize(1024);
199 
200     /**
201      * set M-S : Master: utility1 Slave1: utility2
202      */
203     admin1.addPeer("1", utility2.getClusterKey());
204 
205     admin1.close();
206     admin2.close();
207   }
208 
209   private void putAndReplicateRows() throws Exception {
210     // add rows to Master cluster,
211     Put p;
212 
213     // 100 + 1 row to t1_syncup
214     for (int i = 0; i < NB_ROWS_IN_BATCH; i++) {
215       p = new Put(Bytes.toBytes("row" + i));
216       p.add(famName, qualName, Bytes.toBytes("val" + i));
217       ht1Source.put(p);
218     }
219     p = new Put(Bytes.toBytes("row" + 9999));
220     p.add(noRepfamName, qualName, Bytes.toBytes("val" + 9999));
221     ht1Source.put(p);
222 
223     // 200 + 1 row to t2_syncup
224     for (int i = 0; i < NB_ROWS_IN_BATCH * 2; i++) {
225       p = new Put(Bytes.toBytes("row" + i));
226       p.add(famName, qualName, Bytes.toBytes("val" + i));
227       ht2Source.put(p);
228     }
229     p = new Put(Bytes.toBytes("row" + 9999));
230     p.add(noRepfamName, qualName, Bytes.toBytes("val" + 9999));
231     ht2Source.put(p);
232 
233     // ensure replication completed
234     Thread.sleep(SLEEP_TIME);
235     int rowCount_ht1Source = utility1.countRows(ht1Source);
236     for (int i = 0; i < NB_RETRIES; i++) {
237       int rowCount_ht1TargetAtPeer1 = utility2.countRows(ht1TargetAtPeer1);
238       if (i==NB_RETRIES-1) {
239         assertEquals("t1_syncup has 101 rows on source, and 100 on slave1", rowCount_ht1Source - 1,
240             rowCount_ht1TargetAtPeer1);
241       }
242       if (rowCount_ht1Source - 1 == rowCount_ht1TargetAtPeer1) {
243         break;
244       }
245       Thread.sleep(SLEEP_TIME);
246     }
247 
248     int rowCount_ht2Source = utility1.countRows(ht2Source);
249     for (int i = 0; i < NB_RETRIES; i++) {
250       int rowCount_ht2TargetAtPeer1 = utility2.countRows(ht2TargetAtPeer1);
251       if (i==NB_RETRIES-1) {
252         assertEquals("t2_syncup has 201 rows on source, and 200 on slave1", rowCount_ht2Source - 1,
253             rowCount_ht2TargetAtPeer1);
254       }
255       if (rowCount_ht2Source - 1 == rowCount_ht2TargetAtPeer1) {
256         break;
257       }
258       Thread.sleep(SLEEP_TIME);
259     }
260   }
261 
262   private void mimicSyncUpAfterDelete() throws Exception {
263     utility2.shutdownMiniHBaseCluster();
264 
265     List<Delete> list = new ArrayList<Delete>();
266     // delete half of the rows
267     for (int i = 0; i < NB_ROWS_IN_BATCH / 2; i++) {
268       String rowKey = "row" + i;
269       Delete del = new Delete(rowKey.getBytes());
270       list.add(del);
271     }
272     ht1Source.delete(list);
273 
274     for (int i = 0; i < NB_ROWS_IN_BATCH; i++) {
275       String rowKey = "row" + i;
276       Delete del = new Delete(rowKey.getBytes());
277       list.add(del);
278     }
279     ht2Source.delete(list);
280 
281     int rowCount_ht1Source = utility1.countRows(ht1Source);
282     assertEquals("t1_syncup has 51 rows on source, after remove 50 of the replicated colfam", 51,
283       rowCount_ht1Source);
284 
285     int rowCount_ht2Source = utility1.countRows(ht2Source);
286     assertEquals("t2_syncup has 101 rows on source, after remove 100 of the replicated colfam",
287       101, rowCount_ht2Source);
288 
289     utility1.shutdownMiniHBaseCluster();
290     utility2.restartHBaseCluster(1);
291 
292     Thread.sleep(SLEEP_TIME);
293 
294     // before sync up
295     int rowCount_ht1TargetAtPeer1 = utility2.countRows(ht1TargetAtPeer1);
296     int rowCount_ht2TargetAtPeer1 = utility2.countRows(ht2TargetAtPeer1);
297     assertEquals("@Peer1 t1_syncup should still have 100 rows", 100, rowCount_ht1TargetAtPeer1);
298     assertEquals("@Peer1 t2_syncup should still have 200 rows", 200, rowCount_ht2TargetAtPeer1);
299 
300     // After sync up
301     for (int i = 0; i < NB_RETRIES; i++) {
302       syncUp(utility1);
303       rowCount_ht1TargetAtPeer1 = utility2.countRows(ht1TargetAtPeer1);
304       rowCount_ht2TargetAtPeer1 = utility2.countRows(ht2TargetAtPeer1);
305       if (i == NB_RETRIES - 1) {
306         if (rowCount_ht1TargetAtPeer1 != 50 || rowCount_ht2TargetAtPeer1 != 100) {
307           // syncUP still failed. Let's look at the source in case anything wrong there
308           utility1.restartHBaseCluster(1);
309           rowCount_ht1Source = utility1.countRows(ht1Source);
310           LOG.debug("t1_syncup should have 51 rows at source, and it is " + rowCount_ht1Source);
311           rowCount_ht2Source = utility1.countRows(ht2Source);
312           LOG.debug("t2_syncup should have 101 rows at source, and it is " + rowCount_ht2Source);
313         }
314         assertEquals("@Peer1 t1_syncup should be sync up and have 50 rows", 50,
315           rowCount_ht1TargetAtPeer1);
316         assertEquals("@Peer1 t2_syncup should be sync up and have 100 rows", 100,
317           rowCount_ht2TargetAtPeer1);
318       }
319       if (rowCount_ht1TargetAtPeer1 == 50 && rowCount_ht2TargetAtPeer1 == 100) {
320         LOG.info("SyncUpAfterDelete succeeded at retry = " + i);
321         break;
322       } else {
323         LOG.debug("SyncUpAfterDelete failed at retry = " + i + ", with rowCount_ht1TargetPeer1 ="
324             + rowCount_ht1TargetAtPeer1 + " and rowCount_ht2TargetAtPeer1 ="
325             + rowCount_ht2TargetAtPeer1);
326       }
327       Thread.sleep(SLEEP_TIME);
328     }
329   }
330 
331   private void mimicSyncUpAfterPut() throws Exception {
332     utility1.restartHBaseCluster(1);
333     utility2.shutdownMiniHBaseCluster();
334 
335     Put p;
336     // another 100 + 1 row to t1_syncup
337     // we should see 100 + 2 rows now
338     for (int i = 0; i < NB_ROWS_IN_BATCH; i++) {
339       p = new Put(Bytes.toBytes("row" + i));
340       p.add(famName, qualName, Bytes.toBytes("val" + i));
341       ht1Source.put(p);
342     }
343     p = new Put(Bytes.toBytes("row" + 9998));
344     p.add(noRepfamName, qualName, Bytes.toBytes("val" + 9998));
345     ht1Source.put(p);
346 
347     // another 200 + 1 row to t1_syncup
348     // we should see 200 + 2 rows now
349     for (int i = 0; i < NB_ROWS_IN_BATCH * 2; i++) {
350       p = new Put(Bytes.toBytes("row" + i));
351       p.add(famName, qualName, Bytes.toBytes("val" + i));
352       ht2Source.put(p);
353     }
354     p = new Put(Bytes.toBytes("row" + 9998));
355     p.add(noRepfamName, qualName, Bytes.toBytes("val" + 9998));
356     ht2Source.put(p);
357 
358     int rowCount_ht1Source = utility1.countRows(ht1Source);
359     assertEquals("t1_syncup has 102 rows on source", 102, rowCount_ht1Source);
360     int rowCount_ht2Source = utility1.countRows(ht2Source);
361     assertEquals("t2_syncup has 202 rows on source", 202, rowCount_ht2Source);
362 
363     utility1.shutdownMiniHBaseCluster();
364     utility2.restartHBaseCluster(1);
365 
366     Thread.sleep(SLEEP_TIME);
367 
368     // before sync up
369     int rowCount_ht1TargetAtPeer1 = utility2.countRows(ht1TargetAtPeer1);
370     int rowCount_ht2TargetAtPeer1 = utility2.countRows(ht2TargetAtPeer1);
371     assertEquals("@Peer1 t1_syncup should be NOT sync up and have 50 rows", 50,
372       rowCount_ht1TargetAtPeer1);
373     assertEquals("@Peer1 t2_syncup should be NOT sync up and have 100 rows", 100,
374       rowCount_ht2TargetAtPeer1);
375 
376     // after syun up
377     for (int i = 0; i < NB_RETRIES; i++) {
378       syncUp(utility1);
379       rowCount_ht1TargetAtPeer1 = utility2.countRows(ht1TargetAtPeer1);
380       rowCount_ht2TargetAtPeer1 = utility2.countRows(ht2TargetAtPeer1);
381       if (i == NB_RETRIES - 1) {
382         if (rowCount_ht1TargetAtPeer1 != 100 || rowCount_ht2TargetAtPeer1 != 200) {
383           // syncUP still failed. Let's look at the source in case anything wrong there
384           utility1.restartHBaseCluster(1);
385           rowCount_ht1Source = utility1.countRows(ht1Source);
386           LOG.debug("t1_syncup should have 102 rows at source, and it is " + rowCount_ht1Source);
387           rowCount_ht2Source = utility1.countRows(ht2Source);
388           LOG.debug("t2_syncup should have 202 rows at source, and it is " + rowCount_ht2Source);
389         }
390         assertEquals("@Peer1 t1_syncup should be sync up and have 100 rows", 100,
391           rowCount_ht1TargetAtPeer1);
392         assertEquals("@Peer1 t2_syncup should be sync up and have 200 rows", 200,
393           rowCount_ht2TargetAtPeer1);
394       }
395       if (rowCount_ht1TargetAtPeer1 == 100 && rowCount_ht2TargetAtPeer1 == 200) {
396         LOG.info("SyncUpAfterPut succeeded at retry = " + i);
397         break;
398       } else {
399         LOG.debug("SyncUpAfterPut failed at retry = " + i + ", with rowCount_ht1TargetPeer1 ="
400             + rowCount_ht1TargetAtPeer1 + " and rowCount_ht2TargetAtPeer1 ="
401             + rowCount_ht2TargetAtPeer1);
402       }
403       Thread.sleep(SLEEP_TIME);
404     }
405   }
406 
407   private void syncUp(HBaseTestingUtility ut) throws Exception {
408     ReplicationSyncUp.setConfigure(ut.getConfiguration());
409     String[] arguments = new String[] { null };
410     new ReplicationSyncUp().run(arguments);
411   }
412 
413 }