18
18
package org .apache .hadoop .hbase .client ;
19
19
20
20
import static org .junit .Assert .assertArrayEquals ;
21
+ import static org .junit .Assert .assertTrue ;
22
+ import static org .junit .Assert .fail ;
21
23
22
24
import java .io .IOException ;
25
+ import java .util .concurrent .ExecutorService ;
26
+ import java .util .concurrent .Executors ;
27
+ import java .util .concurrent .ThreadLocalRandom ;
28
+ import java .util .concurrent .TimeUnit ;
23
29
import java .util .stream .Collectors ;
24
30
import java .util .stream .IntStream ;
25
31
import org .apache .hadoop .hbase .HBaseClassTestRule ;
28
34
import org .apache .hadoop .hbase .testclassification .ClientTests ;
29
35
import org .apache .hadoop .hbase .testclassification .MediumTests ;
30
36
import org .apache .hadoop .hbase .util .Bytes ;
37
+ import org .junit .After ;
31
38
import org .junit .AfterClass ;
39
+ import org .junit .Before ;
32
40
import org .junit .BeforeClass ;
33
41
import org .junit .ClassRule ;
34
42
import org .junit .Test ;
35
43
import org .junit .experimental .categories .Category ;
36
44
45
+ import org .apache .hbase .thirdparty .com .google .common .util .concurrent .ThreadFactoryBuilder ;
46
+
37
47
@ Category ({ MediumTests .class , ClientTests .class })
38
48
public class TestBufferedMutator {
39
49
@@ -49,40 +59,85 @@ public class TestBufferedMutator {
49
59
50
60
private static byte [] CQ = Bytes .toBytes ("cq" );
51
61
52
- private static int COUNT = 1024 ;
53
-
54
62
private static byte [] VALUE = new byte [1024 ];
55
63
56
64
@ BeforeClass
57
65
public static void setUp () throws Exception {
58
66
TEST_UTIL .startMiniCluster (1 );
59
- TEST_UTIL . createTable ( TABLE_NAME , CF );
67
+ ThreadLocalRandom . current (). nextBytes ( VALUE );
60
68
}
61
69
62
70
@ AfterClass
63
71
public static void tearDown () throws Exception {
64
72
TEST_UTIL .shutdownMiniCluster ();
65
73
}
66
74
75
+ @ Before
76
+ public void setUpBeforeTest () throws IOException {
77
+ TEST_UTIL .createTable (TABLE_NAME , CF );
78
+ }
79
+
80
+ @ After
81
+ public void tearDownAfterTest () throws IOException {
82
+ TEST_UTIL .deleteTable (TABLE_NAME );
83
+ }
84
+
67
85
@ Test
68
86
public void test () throws Exception {
87
+ int count = 1024 ;
69
88
try (BufferedMutator mutator = TEST_UTIL .getConnection ()
70
89
.getBufferedMutator (new BufferedMutatorParams (TABLE_NAME ).writeBufferSize (64 * 1024 ))) {
71
- mutator .mutate (IntStream .range (0 , COUNT / 2 )
90
+ mutator .mutate (IntStream .range (0 , count / 2 )
72
91
.mapToObj (i -> new Put (Bytes .toBytes (i )).addColumn (CF , CQ , VALUE ))
73
92
.collect (Collectors .toList ()));
74
93
mutator .flush ();
75
- mutator .mutate (IntStream .range (COUNT / 2 , COUNT )
94
+ mutator .mutate (IntStream .range (count / 2 , count )
76
95
.mapToObj (i -> new Put (Bytes .toBytes (i )).addColumn (CF , CQ , VALUE ))
77
96
.collect (Collectors .toList ()));
78
97
mutator .close ();
79
- verifyData ();
98
+ verifyData (count );
99
+ }
100
+ }
101
+
102
+ @ Test
103
+ public void testMultiThread () throws Exception {
104
+ ExecutorService executor =
105
+ Executors .newFixedThreadPool (16 , new ThreadFactoryBuilder ().setDaemon (true ).build ());
106
+ // use a greater count and less write buffer size to trigger auto flush when mutate
107
+ int count = 16384 ;
108
+ try (BufferedMutator mutator = TEST_UTIL .getConnection ()
109
+ .getBufferedMutator (new BufferedMutatorParams (TABLE_NAME ).writeBufferSize (4 * 1024 ))) {
110
+ IntStream .range (0 , count / 2 )
111
+ .mapToObj (i -> new Put (Bytes .toBytes (i )).addColumn (CF , CQ , VALUE ))
112
+ .forEach (put -> executor .execute (() -> {
113
+ try {
114
+ mutator .mutate (put );
115
+ } catch (IOException e ) {
116
+ fail ("failed to mutate: " + e .getMessage ());
117
+ }
118
+ }));
119
+ mutator .flush ();
120
+ IntStream .range (count / 2 , count )
121
+ .mapToObj (i -> new Put (Bytes .toBytes (i )).addColumn (CF , CQ , VALUE ))
122
+ .forEach (put -> executor .execute (() -> {
123
+ try {
124
+ mutator .mutate (put );
125
+ } catch (IOException e ) {
126
+ fail ("failed to mutate: " + e .getMessage ());
127
+ }
128
+ }));
129
+ executor .shutdown ();
130
+ assertTrue (executor .awaitTermination (15 , TimeUnit .SECONDS ));
131
+ mutator .close ();
132
+ } finally {
133
+ executor .shutdownNow ();
80
134
}
135
+ verifyData (count );
81
136
}
82
137
83
- private void verifyData () throws IOException {
138
+ private void verifyData (int count ) throws IOException {
84
139
try (Table table = TEST_UTIL .getConnection ().getTable (TABLE_NAME )) {
85
- for (int i = 0 ; i < COUNT ; i ++) {
140
+ for (int i = 0 ; i < count ; i ++) {
86
141
Result r = table .get (new Get (Bytes .toBytes (i )));
87
142
assertArrayEquals (VALUE , ((Result ) r ).getValue (CF , CQ ));
88
143
}
0 commit comments