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
10 * http://www.apache.org/licenses/LICENSE-2.0
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.
18 package org
.apache
.hadoop
.hbase
;
20 import static org
.junit
.Assert
.assertTrue
;
22 import java
.io
.IOException
;
23 import java
.util
.Arrays
;
24 import java
.util
.List
;
25 import java
.util
.concurrent
.CompletableFuture
;
26 import org
.apache
.hadoop
.hbase
.TestMetaTableAccessor
.SpyingRpcScheduler
;
27 import org
.apache
.hadoop
.hbase
.TestMetaTableAccessor
.SpyingRpcSchedulerFactory
;
28 import org
.apache
.hadoop
.hbase
.client
.AsyncTable
;
29 import org
.apache
.hadoop
.hbase
.client
.Delete
;
30 import org
.apache
.hadoop
.hbase
.client
.Mutation
;
31 import org
.apache
.hadoop
.hbase
.client
.Put
;
32 import org
.apache
.hadoop
.hbase
.client
.RegionInfo
;
33 import org
.apache
.hadoop
.hbase
.client
.RegionInfoBuilder
;
34 import org
.apache
.hadoop
.hbase
.regionserver
.HRegionServer
;
35 import org
.apache
.hadoop
.hbase
.regionserver
.RSRpcServices
;
36 import org
.apache
.hadoop
.hbase
.testclassification
.MediumTests
;
37 import org
.apache
.hadoop
.hbase
.testclassification
.MiscTests
;
38 import org
.apache
.hadoop
.hbase
.util
.Bytes
;
39 import org
.apache
.hadoop
.hbase
.util
.EnvironmentEdgeManager
;
40 import org
.apache
.hadoop
.hbase
.util
.FutureUtils
;
41 import org
.junit
.AfterClass
;
42 import org
.junit
.BeforeClass
;
43 import org
.junit
.ClassRule
;
44 import org
.junit
.Test
;
45 import org
.junit
.experimental
.categories
.Category
;
47 import org
.apache
.hadoop
.hbase
.shaded
.protobuf
.ProtobufUtil
;
48 import org
.apache
.hadoop
.hbase
.shaded
.protobuf
.generated
.ClientProtos
;
49 import org
.apache
.hadoop
.hbase
.shaded
.protobuf
.generated
.MultiRowMutationProtos
.MultiRowMutationService
;
50 import org
.apache
.hadoop
.hbase
.shaded
.protobuf
.generated
.MultiRowMutationProtos
.MutateRowsRequest
;
51 import org
.apache
.hadoop
.hbase
.shaded
.protobuf
.generated
.MultiRowMutationProtos
.MutateRowsResponse
;
53 @Category({ MiscTests
.class, MediumTests
.class })
54 public class TestMetaUpdatesGoToPriorityQueue
{
57 public static final HBaseClassTestRule CLASS_RULE
=
58 HBaseClassTestRule
.forClass(TestMetaUpdatesGoToPriorityQueue
.class);
60 private static final HBaseTestingUtil UTIL
= new HBaseTestingUtil();
63 public static void beforeClass() throws Exception
{
64 // This test has to be end-to-end, and do the verification from the server side
65 UTIL
.getConfiguration().set(RSRpcServices
.REGION_SERVER_RPC_SCHEDULER_FACTORY_CLASS
,
66 SpyingRpcSchedulerFactory
.class.getName());
67 UTIL
.startMiniCluster();
71 public static void afterClass() throws Exception
{
72 UTIL
.shutdownMiniCluster();
75 private void multiMutate(byte[] row
, List
<Mutation
> mutations
) throws IOException
{
76 MutateRowsRequest
.Builder builder
= MutateRowsRequest
.newBuilder();
77 for (Mutation mutation
: mutations
) {
78 if (mutation
instanceof Put
) {
79 builder
.addMutationRequest(
80 ProtobufUtil
.toMutation(ClientProtos
.MutationProto
.MutationType
.PUT
, mutation
));
81 } else if (mutation
instanceof Delete
) {
82 builder
.addMutationRequest(
83 ProtobufUtil
.toMutation(ClientProtos
.MutationProto
.MutationType
.DELETE
, mutation
));
85 throw new DoNotRetryIOException(
86 "multi in MetaEditor doesn't support " + mutation
.getClass().getName());
89 MutateRowsRequest request
= builder
.build();
90 AsyncTable
<?
> table
= UTIL
.getAsyncConnection().getTable(TableName
.META_TABLE_NAME
);
91 CompletableFuture
<MutateRowsResponse
> future
=
92 table
.<MultiRowMutationService
, MutateRowsResponse
> coprocessorService(
93 MultiRowMutationService
::newStub
,
94 (stub
, controller
, done
) -> stub
.mutateRows(controller
, request
, done
), row
);
95 FutureUtils
.get(future
);
99 public void test() throws IOException
, InterruptedException
{
100 TableName tableName
= TableName
.valueOf(getClass().getSimpleName());
101 // create a table and prepare for a manual split
102 UTIL
.createTable(tableName
, "cf1");
103 UTIL
.waitTableAvailable(tableName
);
104 RegionInfo parent
= UTIL
.getAdmin().getRegions(tableName
).get(0);
106 byte[] splitKey
= Bytes
.toBytes("a");
108 RegionInfoBuilder
.newBuilder(parent
.getTable()).setStartKey(parent
.getStartKey())
109 .setEndKey(splitKey
).setSplit(false).setRegionId(rid
).build();
110 RegionInfo splitB
= RegionInfoBuilder
.newBuilder(parent
.getTable()).setStartKey(splitKey
)
111 .setEndKey(parent
.getEndKey()).setSplit(false).setRegionId(rid
).build();
113 // find the meta server
114 SingleProcessHBaseCluster cluster
= UTIL
.getMiniHBaseCluster();
115 int rsIndex
= cluster
.getServerWithMeta();
116 assertTrue(rsIndex
>= 0);
117 HRegionServer rs
= cluster
.getRegionServer(rsIndex
);
118 SpyingRpcScheduler scheduler
= (SpyingRpcScheduler
) rs
.getRpcServer().getScheduler();
119 long prevCalls
= scheduler
.numPriorityCalls
;
120 long time
= EnvironmentEdgeManager
.currentTime();
121 Put putParent
= MetaTableAccessor
.makePutFromRegionInfo(
122 RegionInfoBuilder
.newBuilder(parent
).setOffline(true).setSplit(true).build(), time
);
123 MetaTableAccessor
.addDaughtersToPut(putParent
, splitA
, splitB
);
124 Put putA
= MetaTableAccessor
.makePutFromRegionInfo(splitA
, time
);
125 Put putB
= MetaTableAccessor
.makePutFromRegionInfo(splitB
, time
);
126 multiMutate(putParent
.getRow(), Arrays
.asList(putParent
, putA
, putB
));
128 assertTrue(prevCalls
< scheduler
.numPriorityCalls
);