HBASE-22673 Avoid to expose protobuf stuff in Hbck interface
Signed-off-by: stack <stack@apache.org> Signed-off-by: Duo Zhang <zhangduo@apache.org>
This commit is contained in:
parent
ed2d7e4110
commit
48c623c808
|
@ -23,6 +23,7 @@ import java.util.Map;
|
|||
import java.util.concurrent.Callable;
|
||||
import java.util.stream.Collectors;
|
||||
import org.apache.hadoop.conf.Configuration;
|
||||
import org.apache.hadoop.hbase.ServerName;
|
||||
import org.apache.hadoop.hbase.TableName;
|
||||
import org.apache.hadoop.hbase.ipc.RpcControllerFactory;
|
||||
import org.apache.yetus.audience.InterfaceAudience;
|
||||
|
@ -33,7 +34,6 @@ import org.apache.hbase.thirdparty.com.google.protobuf.ServiceException;
|
|||
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.GetTableStateResponse;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.HbckService.BlockingInterface;
|
||||
|
@ -156,7 +156,7 @@ public class HBaseHbck implements Hbck {
|
|||
}
|
||||
|
||||
@Override
|
||||
public List<Long> scheduleServerCrashProcedure(List<HBaseProtos.ServerName> serverNames)
|
||||
public List<Long> scheduleServerCrashProcedures(List<ServerName> serverNames)
|
||||
throws IOException {
|
||||
try {
|
||||
MasterProtos.ScheduleServerCrashProcedureResponse response =
|
||||
|
@ -171,18 +171,4 @@ public class HBaseHbck implements Hbck {
|
|||
throw new IOException(se);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public Map<String, MasterProtos.RegionErrorType>
|
||||
getFailedSplitMergeLegacyRegions(List<TableName> tableNames) throws IOException {
|
||||
try {
|
||||
MasterProtos.GetFailedSplitMergeLegacyRegionsResponse response =
|
||||
this.hbck.getFailedSplitMergeLegacyRegions(rpcControllerFactory.newController(),
|
||||
RequestConverter.toGetFailedSplitMergeLegacyRegionsRequest(tableNames));
|
||||
return response.getErrorsMap();
|
||||
} catch (ServiceException se) {
|
||||
LOG.debug("get failed split/merge legacy regions failed", se);
|
||||
throw new IOException(se);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
@ -21,11 +21,15 @@ import java.io.Closeable;
|
|||
import java.io.IOException;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
import java.util.stream.Collectors;
|
||||
|
||||
import org.apache.hadoop.hbase.Abortable;
|
||||
import org.apache.hadoop.hbase.HBaseInterfaceAudience;
|
||||
import org.apache.hadoop.hbase.ServerName;
|
||||
import org.apache.hadoop.hbase.TableName;
|
||||
import org.apache.yetus.audience.InterfaceAudience;
|
||||
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos;
|
||||
|
||||
|
@ -105,18 +109,16 @@ public interface Hbck extends Abortable, Closeable {
|
|||
List<Boolean> bypassProcedure(List<Long> pids, long waitTime, boolean override, boolean recursive)
|
||||
throws IOException;
|
||||
|
||||
List<Long> scheduleServerCrashProcedure(List<HBaseProtos.ServerName> serverNames)
|
||||
throws IOException;
|
||||
|
||||
/**
|
||||
* This method is to get the regions which left by failed split/merge procedures for a certain
|
||||
* table. There are two kinds of region this method will return. One is orphan regions left on FS,
|
||||
* which left because split/merge procedure crashed before updating meta. And the other one is
|
||||
* unassigned split daughter region or merged region, which left because split/merge procedure
|
||||
* crashed before assignment.
|
||||
* @param tableName table to check
|
||||
* @return Map of problematic regions
|
||||
* Use {@link #scheduleServerCrashProcedures(List)} instead.
|
||||
* @deprecated since 2.2.1. Will removed in 3.0.0.
|
||||
*/
|
||||
Map<String, MasterProtos.RegionErrorType>
|
||||
getFailedSplitMergeLegacyRegions(List<TableName> tableName) throws IOException;
|
||||
@Deprecated
|
||||
default List<Long> scheduleServerCrashProcedure(List<HBaseProtos.ServerName> serverNames)
|
||||
throws IOException {
|
||||
return scheduleServerCrashProcedures(
|
||||
serverNames.stream().map(ProtobufUtil::toServerName).collect(Collectors.toList()));
|
||||
}
|
||||
|
||||
List<Long> scheduleServerCrashProcedures(List<ServerName> serverNames) throws IOException;
|
||||
}
|
||||
|
|
|
@ -1887,19 +1887,11 @@ public final class RequestConverter {
|
|||
}
|
||||
|
||||
public static MasterProtos.ScheduleServerCrashProcedureRequest
|
||||
toScheduleServerCrashProcedureRequest(List<HBaseProtos.ServerName> serverNames) {
|
||||
MasterProtos.ScheduleServerCrashProcedureRequest.Builder b =
|
||||
toScheduleServerCrashProcedureRequest(List<ServerName> serverNames) {
|
||||
MasterProtos.ScheduleServerCrashProcedureRequest.Builder builder =
|
||||
MasterProtos.ScheduleServerCrashProcedureRequest.newBuilder();
|
||||
return b.addAllServerName(serverNames).build();
|
||||
}
|
||||
|
||||
public static MasterProtos.GetFailedSplitMergeLegacyRegionsRequest
|
||||
toGetFailedSplitMergeLegacyRegionsRequest(List<TableName> tableNames) {
|
||||
MasterProtos.GetFailedSplitMergeLegacyRegionsRequest.Builder b =
|
||||
MasterProtos.GetFailedSplitMergeLegacyRegionsRequest.newBuilder();
|
||||
List<HBaseProtos.TableName> protoTableNames = tableNames.stream()
|
||||
.map(tableName -> ProtobufUtil.toProtoTableName(tableName)).collect(Collectors.toList());
|
||||
return b.addAllTable(protoTableNames).build();
|
||||
serverNames.stream().map(ProtobufUtil::toServerName).forEach(sn -> builder.addServerName(sn));
|
||||
return builder.build();
|
||||
}
|
||||
|
||||
private static List<RegionSpecifier> toEncodedRegionNameRegionSpecifiers(
|
||||
|
|
|
@ -1108,19 +1108,6 @@ message ScheduleServerCrashProcedureResponse {
|
|||
repeated uint64 pid = 1;
|
||||
}
|
||||
|
||||
message GetFailedSplitMergeLegacyRegionsRequest {
|
||||
repeated TableName table = 1;
|
||||
}
|
||||
|
||||
enum RegionErrorType {
|
||||
DAUGHTER_MERGED_REGION_NOT_ONLINE = 0;
|
||||
ORPHAN_REGION_ON_FS = 1;
|
||||
}
|
||||
|
||||
message GetFailedSplitMergeLegacyRegionsResponse {
|
||||
map<string, RegionErrorType> errors = 1;
|
||||
}
|
||||
|
||||
service HbckService {
|
||||
/** Update state of the table in meta only*/
|
||||
rpc SetTableStateInMeta(SetTableStateInMetaRequest)
|
||||
|
@ -1151,7 +1138,4 @@ service HbckService {
|
|||
/** Schedule a ServerCrashProcedure to help recover a crash server */
|
||||
rpc ScheduleServerCrashProcedure(ScheduleServerCrashProcedureRequest)
|
||||
returns(ScheduleServerCrashProcedureResponse);
|
||||
|
||||
rpc getFailedSplitMergeLegacyRegions(GetFailedSplitMergeLegacyRegionsRequest)
|
||||
returns(GetFailedSplitMergeLegacyRegionsResponse);
|
||||
}
|
||||
|
|
|
@ -2528,163 +2528,6 @@ public class MasterRpcServices extends RSRpcServices
|
|||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public MasterProtos.GetFailedSplitMergeLegacyRegionsResponse getFailedSplitMergeLegacyRegions(
|
||||
RpcController controller, MasterProtos.GetFailedSplitMergeLegacyRegionsRequest request)
|
||||
throws ServiceException {
|
||||
List<HBaseProtos.TableName> tables = request.getTableList();
|
||||
|
||||
Map<String, MasterProtos.RegionErrorType> errorRegions = new HashMap<>();
|
||||
try {
|
||||
for (HBaseProtos.TableName tableName : tables) {
|
||||
errorRegions.putAll(getFailedSplitMergeLegacyRegions(ProtobufUtil.toTableName(tableName)));
|
||||
}
|
||||
} catch (IOException e) {
|
||||
throw new ServiceException(e);
|
||||
}
|
||||
return MasterProtos.GetFailedSplitMergeLegacyRegionsResponse.newBuilder()
|
||||
.putAllErrors(errorRegions).build();
|
||||
}
|
||||
|
||||
private Map<String, MasterProtos.RegionErrorType>
|
||||
getFailedSplitMergeLegacyRegions(TableName tableName) throws IOException {
|
||||
if (!MetaTableAccessor.tableExists(master.getConnection(), tableName)) {
|
||||
throw new IOException("table " + tableName.getNameAsString() + " doesn't exist");
|
||||
}
|
||||
if (!MetaTableAccessor.getTableState(master.getConnection(), tableName).isEnabled()) {
|
||||
throw new IOException(
|
||||
"table " + tableName.getNameAsString() + " is not enabled yet");
|
||||
}
|
||||
final Map<String, MasterProtos.RegionErrorType> problemRegions = new HashMap<>();
|
||||
|
||||
// Case 1. find orphan region on fs
|
||||
// orphan regions may due to a failed split region procedure, which daughter regions are created
|
||||
// then the procedure is aborted. Or merged region is created then the procedure is aborted.
|
||||
List<String> orphanRegions = findOrphanRegionOnFS(tableName);
|
||||
orphanRegions.stream().forEach(
|
||||
region -> problemRegions.put(region, MasterProtos.RegionErrorType.ORPHAN_REGION_ON_FS));
|
||||
|
||||
// Case 2. find unassigned daughter regions or merged regions
|
||||
List<String> unassignedDaughterOrMergedRegions =
|
||||
findUnassignedDaughterOrMergedRegions(tableName);
|
||||
unassignedDaughterOrMergedRegions.stream().forEach(region -> problemRegions.put(region,
|
||||
MasterProtos.RegionErrorType.DAUGHTER_MERGED_REGION_NOT_ONLINE));
|
||||
|
||||
// if these regions in problemRegions are currently handled by SplitTableRegionProcedure or
|
||||
// MergeTableRegionsProcedure, we should remove them from this map
|
||||
master.getProcedures().stream().filter(p -> !(p.isFinished() || p.isBypass())).forEach(p -> {
|
||||
if (p instanceof SplitTableRegionProcedure) {
|
||||
problemRegions
|
||||
.remove(((SplitTableRegionProcedure) p).getDaughterOneRI().getRegionNameAsString());
|
||||
problemRegions
|
||||
.remove(((SplitTableRegionProcedure) p).getDaughterTwoRI().getRegionNameAsString());
|
||||
} else if (p instanceof MergeTableRegionsProcedure) {
|
||||
problemRegions
|
||||
.remove(((MergeTableRegionsProcedure) p).getMergedRegion().getRegionNameAsString());
|
||||
}
|
||||
});
|
||||
|
||||
// check if regions are still problematic now
|
||||
checkRegionStillProblematic(problemRegions, tableName);
|
||||
return problemRegions;
|
||||
}
|
||||
|
||||
|
||||
private void checkRegionStillProblematic(
|
||||
Map<String, MasterProtos.RegionErrorType> problemRegions, TableName tableName)
|
||||
throws IOException {
|
||||
Iterator<Map.Entry<String, MasterProtos.RegionErrorType>> iterator =
|
||||
problemRegions.entrySet().iterator();
|
||||
while (iterator.hasNext()) {
|
||||
Map.Entry<String, MasterProtos.RegionErrorType> entry = iterator.next();
|
||||
Result r = MetaTableAccessor.getRegionResult(master.getConnection(),
|
||||
Bytes.toBytesBinary(entry.getKey()));
|
||||
switch (entry.getValue()) {
|
||||
case ORPHAN_REGION_ON_FS:
|
||||
// region is build for this directory, it is not a problematic region any more
|
||||
if (r != null) {
|
||||
iterator.remove();
|
||||
}
|
||||
break;
|
||||
case DAUGHTER_MERGED_REGION_NOT_ONLINE:
|
||||
RegionState.State state = RegionStateStore.getRegionState(r, 0);
|
||||
if (!state.matches(RegionState.State.CLOSED, RegionState.State.SPLITTING_NEW,
|
||||
RegionState.State.MERGED)) {
|
||||
iterator.remove();
|
||||
}
|
||||
break;
|
||||
default:
|
||||
throw new IOException("there should be no problematic region of this type");
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
private List<String> findUnassignedDaughterOrMergedRegions(TableName tableName)
|
||||
throws IOException {
|
||||
Set<String> checkRegions = new HashSet<>();
|
||||
Map<String, RegionState.State> regionStates = new HashMap<>();
|
||||
Map<String, RegionInfo> regionInfos = new HashMap<>();
|
||||
|
||||
MetaTableAccessor.scanMeta(master.getConnection(), tableName,
|
||||
MetaTableAccessor.QueryType.REGION, Integer.MAX_VALUE, r -> {
|
||||
RegionInfo regionInfo = MetaTableAccessor.getRegionInfo(r);
|
||||
regionInfos.put(regionInfo.getRegionNameAsString(), regionInfo);
|
||||
RegionState.State state = RegionStateStore.getRegionState(r, 0);
|
||||
regionStates.put(regionInfo.getEncodedName(), state);
|
||||
if (regionInfo.isSplitParent()) {
|
||||
PairOfSameType<RegionInfo> daughters = MetaTableAccessor.getDaughterRegions(r);
|
||||
checkRegions.add(daughters.getFirst().getRegionNameAsString());
|
||||
checkRegions.add(daughters.getSecond().getRegionNameAsString());
|
||||
} else if (r.getValue(HConstants.CATALOG_FAMILY, HConstants.MERGEA_QUALIFIER) != null
|
||||
|| r.getValue(HConstants.CATALOG_FAMILY, HConstants.MERGEB_QUALIFIER) != null) {
|
||||
checkRegions.add(regionInfo.getRegionNameAsString());
|
||||
}
|
||||
return true;
|
||||
});
|
||||
|
||||
// find unassigned merged or split daughter region
|
||||
return checkRegions.stream().map(regionName -> regionInfos.get(regionName))
|
||||
.filter(regionInfo -> !regionInfo.isSplitParent())
|
||||
.filter(regionInfo -> !regionStates.get(regionInfo.getEncodedName())
|
||||
.matches(RegionState.State.OPEN))
|
||||
.map(regionInfo -> regionInfo.getRegionNameAsString()).collect(Collectors.toList());
|
||||
}
|
||||
|
||||
private List<String> findOrphanRegionOnFS(TableName tableName) throws IOException {
|
||||
// get available regions from meta, merged region should be consider available
|
||||
HashSet<String> regionsInMeta = new HashSet<>();
|
||||
MetaTableAccessor.scanMeta(master.getConnection(), tableName,
|
||||
MetaTableAccessor.QueryType.REGION, Integer.MAX_VALUE, r -> {
|
||||
RegionInfo regionInfo = MetaTableAccessor.getRegionInfo(r);
|
||||
regionsInMeta.add(regionInfo.getEncodedName());
|
||||
if (r.getValue(HConstants.CATALOG_FAMILY, HConstants.MERGEA_QUALIFIER) != null
|
||||
|| r.getValue(HConstants.CATALOG_FAMILY, HConstants.MERGEB_QUALIFIER) != null) {
|
||||
PairOfSameType<RegionInfo> mergedRegions = MetaTableAccessor.getMergeRegions(r);
|
||||
regionsInMeta.add(mergedRegions.getFirst().getEncodedName());
|
||||
regionsInMeta.add(mergedRegions.getSecond().getEncodedName());
|
||||
}
|
||||
return true;
|
||||
});
|
||||
// get regionInfo from fs
|
||||
Path tableDir = FSUtils.getTableDir(master.getMasterFileSystem().getRootDir(), tableName);
|
||||
FileStatus[] regions =
|
||||
master.getFileSystem().listStatus(tableDir, path -> !path.getName().startsWith("."));
|
||||
HashMap<String, String> regionNames = new HashMap<>();
|
||||
for (FileStatus region : regions) {
|
||||
RegionInfo regionInfo =
|
||||
HRegionFileSystem.loadRegionInfoFileContent(master.getFileSystem(), region.getPath());
|
||||
regionNames.put(regionInfo.getEncodedName(), regionInfo.getRegionNameAsString());
|
||||
}
|
||||
Iterator<Map.Entry<String, String>> regionIterator = regionNames.entrySet().iterator();
|
||||
while (regionIterator.hasNext()) {
|
||||
Map.Entry<String, String> region = regionIterator.next();
|
||||
if (regionsInMeta.contains(region.getKey())) {
|
||||
regionIterator.remove();
|
||||
}
|
||||
}
|
||||
return new ArrayList<>(regionNames.values());
|
||||
}
|
||||
|
||||
@Override
|
||||
public SwitchRpcThrottleResponse switchRpcThrottle(RpcController controller,
|
||||
SwitchRpcThrottleRequest request) throws ServiceException {
|
||||
|
|
|
@ -18,16 +18,13 @@
|
|||
package org.apache.hadoop.hbase.client;
|
||||
|
||||
import static org.junit.Assert.assertEquals;
|
||||
import static org.junit.Assert.assertNotNull;
|
||||
import static org.junit.Assert.assertTrue;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.Arrays;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
import java.util.Optional;
|
||||
import java.util.concurrent.CountDownLatch;
|
||||
import java.util.concurrent.TimeUnit;
|
||||
import java.util.stream.Collectors;
|
||||
import org.apache.hadoop.hbase.Coprocessor;
|
||||
import org.apache.hadoop.hbase.CoprocessorEnvironment;
|
||||
|
@ -39,10 +36,7 @@ import org.apache.hadoop.hbase.coprocessor.MasterCoprocessor;
|
|||
import org.apache.hadoop.hbase.coprocessor.MasterCoprocessorEnvironment;
|
||||
import org.apache.hadoop.hbase.coprocessor.MasterObserver;
|
||||
import org.apache.hadoop.hbase.coprocessor.ObserverContext;
|
||||
import org.apache.hadoop.hbase.master.HMaster;
|
||||
import org.apache.hadoop.hbase.master.RegionState;
|
||||
import org.apache.hadoop.hbase.master.assignment.MergeTableRegionsProcedure;
|
||||
import org.apache.hadoop.hbase.master.assignment.SplitTableRegionProcedure;
|
||||
import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
|
||||
import org.apache.hadoop.hbase.master.procedure.TableProcedureInterface;
|
||||
import org.apache.hadoop.hbase.procedure2.Procedure;
|
||||
|
@ -54,7 +48,6 @@ import org.apache.hadoop.hbase.testclassification.ClientTests;
|
|||
import org.apache.hadoop.hbase.testclassification.LargeTests;
|
||||
import org.apache.hadoop.hbase.util.Bytes;
|
||||
import org.junit.AfterClass;
|
||||
import org.junit.Assert;
|
||||
import org.junit.Before;
|
||||
import org.junit.BeforeClass;
|
||||
import org.junit.ClassRule;
|
||||
|
@ -72,7 +65,6 @@ import org.slf4j.LoggerFactory;
|
|||
import org.apache.hbase.thirdparty.com.google.common.io.Closeables;
|
||||
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos;
|
||||
|
||||
/**
|
||||
* Class to test HBaseHbck. Spins up the minicluster once at test start and then takes it down
|
||||
|
@ -226,102 +218,6 @@ public class TestHbck {
|
|||
}
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testRecoverMergeAfterMetaUpdated() throws Exception {
|
||||
String testTable = async ? "mergeTestAsync" : "mergeTestSync";
|
||||
TEST_UTIL.createMultiRegionTable(TableName.valueOf(testTable), Bytes.toBytes("family1"), 5);
|
||||
TEST_UTIL.loadTable(TEST_UTIL.getConnection().getTable(TableName.valueOf(testTable)),
|
||||
Bytes.toBytes("family1"), true);
|
||||
HMaster master = TEST_UTIL.getHBaseCluster().getMaster();
|
||||
Hbck hbck = getHbck();
|
||||
FailingMergeAfterMetaUpdatedMasterObserver observer = master.getMasterCoprocessorHost()
|
||||
.findCoprocessor(FailingMergeAfterMetaUpdatedMasterObserver.class);
|
||||
try (Admin admin = TEST_UTIL.getConnection().getAdmin()) {
|
||||
List<RegionInfo> regions = admin.getRegions(TableName.valueOf(testTable));
|
||||
admin.mergeRegionsAsync(regions.get(0).getRegionName(), regions.get(1).getRegionName(), true);
|
||||
assertNotNull(observer);
|
||||
observer.latch.await(5000, TimeUnit.MILLISECONDS);
|
||||
Map<String, MasterProtos.RegionErrorType> result =
|
||||
hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(0, result.size());
|
||||
Optional<Procedure<?>> procedure = TEST_UTIL.getHBaseCluster().getMaster().getProcedures()
|
||||
.stream().filter(p -> p instanceof MergeTableRegionsProcedure).findAny();
|
||||
Assert.assertTrue(procedure.isPresent());
|
||||
hbck.bypassProcedure(Arrays.asList(procedure.get().getProcId()), 5, true, false);
|
||||
result = hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(1, result.size());
|
||||
hbck.assigns(Arrays.asList(result.keySet().toArray(new String[0])).stream()
|
||||
.map(regionName -> regionName.split("\\.")[1]).collect(Collectors.toList()));
|
||||
ProcedureTestingUtility.waitAllProcedures(master.getMasterProcedureExecutor());
|
||||
// now the state should be fixed
|
||||
result = hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(0, result.size());
|
||||
} catch (InterruptedException ie) {
|
||||
throw new IOException(ie);
|
||||
} finally {
|
||||
observer.resetLatch();
|
||||
}
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testRecoverSplitAfterMetaUpdated() throws Exception {
|
||||
String testTable = async ? "splitTestAsync" : "splitTestSync";
|
||||
TEST_UTIL.createMultiRegionTable(TableName.valueOf(testTable), Bytes.toBytes("family1"), 5);
|
||||
HMaster master = TEST_UTIL.getHBaseCluster().getMaster();
|
||||
Hbck hbck = getHbck();
|
||||
FailingSplitAfterMetaUpdatedMasterObserver observer = master.getMasterCoprocessorHost()
|
||||
.findCoprocessor(FailingSplitAfterMetaUpdatedMasterObserver.class);
|
||||
assertNotNull(observer);
|
||||
try {
|
||||
AsyncAdmin admin = TEST_UTIL.getAsyncConnection().getAdmin();
|
||||
byte[] splitKey = Bytes.toBytes("bcd");
|
||||
admin.split(TableName.valueOf(testTable), splitKey);
|
||||
observer.latch.await(5000, TimeUnit.MILLISECONDS);
|
||||
Map<String, MasterProtos.RegionErrorType> result =
|
||||
hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
// since there is a split procedure work on the region, thus this check should return a empty
|
||||
// map.
|
||||
Assert.assertEquals(0, result.size());
|
||||
Optional<Procedure<?>> procedure = TEST_UTIL.getHBaseCluster().getMaster().getProcedures()
|
||||
.stream().filter(p -> p instanceof SplitTableRegionProcedure).findAny();
|
||||
Assert.assertTrue(procedure.isPresent());
|
||||
hbck.bypassProcedure(Arrays.asList(procedure.get().getProcId()), 5, true, false);
|
||||
result = hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(2, result.size());
|
||||
hbck.assigns(Arrays.asList(result.keySet().toArray(new String[0])).stream()
|
||||
.map(regionName -> regionName.split("\\.")[1]).collect(Collectors.toList()));
|
||||
ProcedureTestingUtility.waitAllProcedures(master.getMasterProcedureExecutor());
|
||||
// now the state should be fixed
|
||||
result = hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(0, result.size());
|
||||
|
||||
//split one of the daughter region again
|
||||
observer.resetLatch();
|
||||
byte[] splitKey2 = Bytes.toBytes("bcde");
|
||||
|
||||
admin.split(TableName.valueOf(testTable), splitKey2);
|
||||
observer.latch.await(5000, TimeUnit.MILLISECONDS);
|
||||
|
||||
procedure = TEST_UTIL.getHBaseCluster().getMaster().getProcedures()
|
||||
.stream().filter(p -> p instanceof SplitTableRegionProcedure).findAny();
|
||||
Assert.assertTrue(procedure.isPresent());
|
||||
hbck.bypassProcedure(Arrays.asList(procedure.get().getProcId()), 5, true, false);
|
||||
result = hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(2, result.size());
|
||||
hbck.assigns(Arrays.asList(result.keySet().toArray(new String[0])).stream()
|
||||
.map(regionName -> regionName.split("\\.")[1]).collect(Collectors.toList()));
|
||||
ProcedureTestingUtility.waitAllProcedures(master.getMasterProcedureExecutor());
|
||||
// now the state should be fixed
|
||||
result = hbck.getFailedSplitMergeLegacyRegions(Arrays.asList(TableName.valueOf(testTable)));
|
||||
Assert.assertEquals(0, result.size());
|
||||
} catch (InterruptedException ie) {
|
||||
throw new IOException(ie);
|
||||
} finally {
|
||||
observer.resetLatch();
|
||||
}
|
||||
}
|
||||
|
||||
|
||||
@Test
|
||||
public void testScheduleSCP() throws Exception {
|
||||
HRegionServer testRs = TEST_UTIL.getRSForFirstRegionInTable(TABLE_NAME);
|
||||
|
|
Loading…
Reference in New Issue