HBASE-26245 Store region server list in master local region (#4136)
Signed-off-by: Andrew Purtell <apurtell@apache.org> (cherry picked from commitbb1bbddf74
) (cherry picked from commit27111421d9
)
This commit is contained in:
parent
08ecae97e3
commit
75c81c7338
|
@ -402,6 +402,8 @@ public class HMaster extends HRegionServer implements MasterServices {
|
|||
// the master local storage to store procedure data, meta region locations, etc.
|
||||
private MasterRegion masterRegion;
|
||||
|
||||
private RegionServerList rsListStorage;
|
||||
|
||||
// handle table states
|
||||
private TableStateManager tableStateManager;
|
||||
|
||||
|
@ -896,14 +898,19 @@ public class HMaster extends HRegionServer implements MasterServices {
|
|||
}
|
||||
|
||||
status.setStatus("Initialize ServerManager and schedule SCP for crash servers");
|
||||
this.serverManager = createServerManager(this);
|
||||
// The below two managers must be created before loading procedures, as they will be used during
|
||||
// loading.
|
||||
// initialize master local region
|
||||
masterRegion = MasterRegionFactory.create(this);
|
||||
rsListStorage = new MasterRegionServerList(masterRegion, this);
|
||||
|
||||
this.serverManager = createServerManager(this, rsListStorage);
|
||||
if (!conf.getBoolean(HBASE_SPLIT_WAL_COORDINATED_BY_ZK,
|
||||
DEFAULT_HBASE_SPLIT_COORDINATED_BY_ZK)) {
|
||||
this.splitWALManager = new SplitWALManager(this);
|
||||
}
|
||||
|
||||
// initialize master local region
|
||||
masterRegion = MasterRegionFactory.create(this);
|
||||
|
||||
|
||||
tryMigrateMetaLocationsFromZooKeeper();
|
||||
|
||||
|
@ -932,7 +939,8 @@ public class HMaster extends HRegionServer implements MasterServices {
|
|||
this.regionServerTracker.upgrade(
|
||||
procsByType.getOrDefault(ServerCrashProcedure.class, Collections.emptyList()).stream()
|
||||
.map(p -> (ServerCrashProcedure) p).map(p -> p.getServerName()).collect(Collectors.toSet()),
|
||||
walManager.getLiveServersFromWALDir(), walManager.getSplittingServersFromWALDir());
|
||||
Sets.union(rsListStorage.getAll(), walManager.getLiveServersFromWALDir()),
|
||||
walManager.getSplittingServersFromWALDir());
|
||||
// This manager will be started AFTER hbase:meta is confirmed on line.
|
||||
// hbase.mirror.table.state.to.zookeeper is so hbase1 clients can connect. They read table
|
||||
// state from zookeeper while hbase2 reads it from hbase:meta. Disable if no hbase1 clients.
|
||||
|
@ -1376,11 +1384,12 @@ public class HMaster extends HRegionServer implements MasterServices {
|
|||
* </p>
|
||||
*/
|
||||
@InterfaceAudience.Private
|
||||
protected ServerManager createServerManager(final MasterServices master) throws IOException {
|
||||
protected ServerManager createServerManager(MasterServices master,
|
||||
RegionServerList storage) throws IOException {
|
||||
// We put this out here in a method so can do a Mockito.spy and stub it out
|
||||
// w/ a mocked up ServerManager.
|
||||
setupClusterConnection();
|
||||
return new ServerManager(master);
|
||||
return new ServerManager(master, storage);
|
||||
}
|
||||
|
||||
private void waitForRegionServers(final MonitoredTask status)
|
||||
|
|
|
@ -0,0 +1,111 @@
|
|||
/**
|
||||
* Licensed to the Apache Software Foundation (ASF) under one
|
||||
* or more contributor license agreements. See the NOTICE file
|
||||
* distributed with this work for additional information
|
||||
* regarding copyright ownership. The ASF licenses this file
|
||||
* to you under the Apache License, Version 2.0 (the
|
||||
* "License"); you may not use this file except in compliance
|
||||
* with the License. You may obtain a copy of the License at
|
||||
*
|
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
*
|
||||
* Unless required by applicable law or agreed to in writing, software
|
||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.hadoop.hbase.master;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.UncheckedIOException;
|
||||
import java.util.HashSet;
|
||||
import java.util.Set;
|
||||
import org.apache.hadoop.hbase.Abortable;
|
||||
import org.apache.hadoop.hbase.HConstants;
|
||||
import org.apache.hadoop.hbase.ServerName;
|
||||
import org.apache.hadoop.hbase.client.Delete;
|
||||
import org.apache.hadoop.hbase.client.Put;
|
||||
import org.apache.hadoop.hbase.client.Result;
|
||||
import org.apache.hadoop.hbase.client.ResultScanner;
|
||||
import org.apache.hadoop.hbase.client.Scan;
|
||||
import org.apache.hadoop.hbase.log.HBaseMarkers;
|
||||
import org.apache.hadoop.hbase.master.assignment.ServerState;
|
||||
import org.apache.hadoop.hbase.master.region.MasterRegion;
|
||||
import org.apache.hadoop.hbase.master.region.MasterRegionFactory;
|
||||
import org.apache.hadoop.hbase.util.Bytes;
|
||||
import org.apache.yetus.audience.InterfaceAudience;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
/**
|
||||
* {@link MasterRegion} based {@link RegionServerList}.
|
||||
* <p/>
|
||||
* This is useful when we want to restart a cluster with only the data on file system, as when
|
||||
* restarting, we need to get the previous live region servers for scheduling SCP. Before we have
|
||||
* this class, we need to scan the WAL directory on WAL file system to find out the previous live
|
||||
* region servers, which means we can not restart a cluster without the previous WAL file system,
|
||||
* even if we have flushed all the data.
|
||||
* <p/>
|
||||
* Please see HBASE-26245 for more details.
|
||||
*/
|
||||
@InterfaceAudience.Private
|
||||
public class MasterRegionServerList implements RegionServerList {
|
||||
|
||||
private static final Logger LOG = LoggerFactory.getLogger(MasterRegionServerList.class);
|
||||
|
||||
private final MasterRegion region;
|
||||
|
||||
private final Abortable abortable;
|
||||
|
||||
public MasterRegionServerList(MasterRegion region, Abortable abortable) {
|
||||
this.region = region;
|
||||
this.abortable = abortable;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void started(ServerName sn) {
|
||||
Put put =
|
||||
new Put(Bytes.toBytes(sn.getServerName())).addColumn(MasterRegionFactory.REGION_SERVER_FAMILY,
|
||||
HConstants.STATE_QUALIFIER, Bytes.toBytes(ServerState.ONLINE.name()));
|
||||
try {
|
||||
region.update(r -> r.put(put));
|
||||
} catch (IOException e) {
|
||||
LOG.error(HBaseMarkers.FATAL, "Failed to record region server {} as started, aborting...", sn,
|
||||
e);
|
||||
abortable.abort("Failed to record region server as started");
|
||||
throw new UncheckedIOException(e);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void expired(ServerName sn) {
|
||||
Delete delete = new Delete(Bytes.toBytes(sn.getServerName()))
|
||||
.addFamily(MasterRegionFactory.REGION_SERVER_FAMILY);
|
||||
try {
|
||||
region.update(r -> r.delete(delete));
|
||||
} catch (IOException e) {
|
||||
LOG.error(HBaseMarkers.FATAL, "Failed to record region server {} as expired, aborting...", sn,
|
||||
e);
|
||||
abortable.abort("Failed to record region server as expired");
|
||||
throw new UncheckedIOException(e);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public Set<ServerName> getAll() throws IOException {
|
||||
Set<ServerName> rsList = new HashSet<>();
|
||||
try (ResultScanner scanner =
|
||||
region.getScanner(new Scan().addFamily(MasterRegionFactory.REGION_SERVER_FAMILY))) {
|
||||
for (;;) {
|
||||
Result result = scanner.next();
|
||||
if (result == null) {
|
||||
break;
|
||||
}
|
||||
rsList.add(ServerName.valueOf(Bytes.toString(result.getRow())));
|
||||
}
|
||||
}
|
||||
return rsList;
|
||||
}
|
||||
|
||||
}
|
|
@ -157,8 +157,6 @@ public class MasterWalManager {
|
|||
|
||||
/**
|
||||
* Get Servernames which are currently splitting; paths have a '-splitting' suffix.
|
||||
* @return ServerName
|
||||
* @throws IOException IOException
|
||||
*/
|
||||
public Set<ServerName> getSplittingServersFromWALDir() throws IOException {
|
||||
return getServerNamesFromWALDirPath(
|
||||
|
@ -168,8 +166,6 @@ public class MasterWalManager {
|
|||
/**
|
||||
* Get Servernames that COULD BE 'alive'; excludes those that have a '-splitting' suffix as these
|
||||
* are already being split -- they cannot be 'alive'.
|
||||
* @return ServerName
|
||||
* @throws IOException IOException
|
||||
*/
|
||||
public Set<ServerName> getLiveServersFromWALDir() throws IOException {
|
||||
return getServerNamesFromWALDirPath(
|
||||
|
|
|
@ -0,0 +1,47 @@
|
|||
/**
|
||||
* Licensed to the Apache Software Foundation (ASF) under one
|
||||
* or more contributor license agreements. See the NOTICE file
|
||||
* distributed with this work for additional information
|
||||
* regarding copyright ownership. The ASF licenses this file
|
||||
* to you under the Apache License, Version 2.0 (the
|
||||
* "License"); you may not use this file except in compliance
|
||||
* with the License. You may obtain a copy of the License at
|
||||
*
|
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
*
|
||||
* Unless required by applicable law or agreed to in writing, software
|
||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.hadoop.hbase.master;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.Set;
|
||||
import org.apache.hadoop.hbase.ServerName;
|
||||
import org.apache.yetus.audience.InterfaceAudience;
|
||||
|
||||
/**
|
||||
* For storing the region server list.
|
||||
* <p/>
|
||||
* Mainly be used when restarting master, to load the previous active region server list.
|
||||
*/
|
||||
@InterfaceAudience.Private
|
||||
public interface RegionServerList {
|
||||
|
||||
/**
|
||||
* Called when a region server join the cluster.
|
||||
*/
|
||||
void started(ServerName sn);
|
||||
|
||||
/**
|
||||
* Called when a region server is dead.
|
||||
*/
|
||||
void expired(ServerName sn);
|
||||
|
||||
/**
|
||||
* Get all live region servers.
|
||||
*/
|
||||
Set<ServerName> getAll() throws IOException;
|
||||
}
|
|
@ -115,22 +115,22 @@ public class RegionServerTracker extends ZKListener {
|
|||
* {@link ServerManager#findDeadServersAndProcess(Set, Set)}, we call it here under the lock
|
||||
* protection to prevent concurrency issues with server expiration operation.
|
||||
* @param deadServersFromPE the region servers which already have SCP associated.
|
||||
* @param liveServersFromWALDir the live region servers from wal directory.
|
||||
* @param liveServersBeforeRestart the live region servers we recorded before master restarts.
|
||||
* @param splittingServersFromWALDir Servers whose WALs are being actively 'split'.
|
||||
*/
|
||||
public void upgrade(Set<ServerName> deadServersFromPE, Set<ServerName> liveServersFromWALDir,
|
||||
public void upgrade(Set<ServerName> deadServersFromPE, Set<ServerName> liveServersBeforeRestart,
|
||||
Set<ServerName> splittingServersFromWALDir) throws KeeperException, IOException {
|
||||
LOG.info(
|
||||
"Upgrading RegionServerTracker to active master mode; {} have existing" +
|
||||
"ServerCrashProcedures, {} possibly 'live' servers, and {} 'splitting'.",
|
||||
deadServersFromPE.size(), liveServersFromWALDir.size(), splittingServersFromWALDir.size());
|
||||
deadServersFromPE.size(), liveServersBeforeRestart.size(), splittingServersFromWALDir.size());
|
||||
// deadServersFromPE is made from a list of outstanding ServerCrashProcedures.
|
||||
// splittingServersFromWALDir are being actively split -- the directory in the FS ends in
|
||||
// '-SPLITTING'. Each splitting server should have a corresponding SCP. Log if not.
|
||||
splittingServersFromWALDir.stream().filter(s -> !deadServersFromPE.contains(s)).
|
||||
forEach(s -> LOG.error("{} has no matching ServerCrashProcedure", s));
|
||||
// create ServerNode for all possible live servers from wal directory
|
||||
liveServersFromWALDir
|
||||
liveServersBeforeRestart
|
||||
.forEach(sn -> server.getAssignmentManager().getRegionStates().getOrCreateServer(sn));
|
||||
ServerManager serverManager = server.getServerManager();
|
||||
synchronized (this) {
|
||||
|
@ -142,7 +142,7 @@ public class RegionServerTracker extends ZKListener {
|
|||
info.getVersionInfo().getVersion()) : ServerMetricsBuilder.of(serverName);
|
||||
serverManager.checkAndRecordNewServer(serverName, serverMetrics);
|
||||
}
|
||||
serverManager.findDeadServersAndProcess(deadServersFromPE, liveServersFromWALDir);
|
||||
serverManager.findDeadServersAndProcess(deadServersFromPE, liveServersBeforeRestart);
|
||||
active = true;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -139,6 +139,7 @@ public class ServerManager {
|
|||
|
||||
private final MasterServices master;
|
||||
private final ClusterConnection connection;
|
||||
private final RegionServerList storage;
|
||||
|
||||
private final DeadServer deadservers = new DeadServer();
|
||||
|
||||
|
@ -153,8 +154,9 @@ public class ServerManager {
|
|||
/**
|
||||
* Constructor.
|
||||
*/
|
||||
public ServerManager(final MasterServices master) {
|
||||
public ServerManager(final MasterServices master, RegionServerList storage) {
|
||||
this.master = master;
|
||||
this.storage = storage;
|
||||
Configuration c = master.getConfiguration();
|
||||
maxSkew = c.getLong("hbase.master.maxclockskew", 30000);
|
||||
warningSkew = c.getLong("hbase.master.warningclockskew", 10000);
|
||||
|
@ -185,7 +187,6 @@ public class ServerManager {
|
|||
* @param version the version of the new regionserver, could contain strings like "SNAPSHOT"
|
||||
* @param ia the InetAddress from which request is received
|
||||
* @return The ServerName we know this server as.
|
||||
* @throws IOException
|
||||
*/
|
||||
ServerName regionServerStartup(RegionServerStartupRequest request, int versionNumber,
|
||||
String version, InetAddress ia) throws IOException {
|
||||
|
@ -206,13 +207,12 @@ public class ServerManager {
|
|||
LOG.warn(
|
||||
"THIS SHOULD NOT HAPPEN, RegionServerStartup" + " could not record the server: " + sn);
|
||||
}
|
||||
storage.started(sn);
|
||||
return sn;
|
||||
}
|
||||
|
||||
/**
|
||||
* Updates last flushed sequence Ids for the regions on server sn
|
||||
* @param sn
|
||||
* @param hsl
|
||||
*/
|
||||
private void updateLastFlushedSequenceIds(ServerName sn, ServerMetrics hsl) {
|
||||
for (Entry<byte[], RegionMetrics> entry : hsl.getRegionMetrics().entrySet()) {
|
||||
|
@ -581,6 +581,7 @@ public class ServerManager {
|
|||
}
|
||||
LOG.info("Processing expiration of " + serverName + " on " + this.master.getServerName());
|
||||
long pid = master.getAssignmentManager().submitServerCrash(serverName, true, force);
|
||||
storage.expired(serverName);
|
||||
// Tell our listeners that a server was removed
|
||||
if (!this.listeners.isEmpty()) {
|
||||
this.listeners.stream().forEach(l -> l.serverRemoved(serverName));
|
||||
|
|
|
@ -87,12 +87,16 @@ public final class MasterRegionFactory {
|
|||
|
||||
public static final byte[] PROC_FAMILY = Bytes.toBytes("proc");
|
||||
|
||||
public static final byte[] REGION_SERVER_FAMILY = Bytes.toBytes("rs");
|
||||
|
||||
private static final TableDescriptor TABLE_DESC = TableDescriptorBuilder.newBuilder(TABLE_NAME)
|
||||
.setColumnFamily(ColumnFamilyDescriptorBuilder.newBuilder(HConstants.CATALOG_FAMILY)
|
||||
.setMaxVersions(HConstants.DEFAULT_HBASE_META_VERSIONS).setInMemory(true)
|
||||
.setBlocksize(HConstants.DEFAULT_HBASE_META_BLOCK_SIZE).setBloomFilterType(BloomType.ROWCOL)
|
||||
.setDataBlockEncoding(DataBlockEncoding.ROW_INDEX_V1).build())
|
||||
.setColumnFamily(ColumnFamilyDescriptorBuilder.of(PROC_FAMILY)).build();
|
||||
.setColumnFamily(ColumnFamilyDescriptorBuilder.of(PROC_FAMILY))
|
||||
.setColumnFamily(ColumnFamilyDescriptorBuilder.of(REGION_SERVER_FAMILY))
|
||||
.build();
|
||||
|
||||
private static TableDescriptor withTrackerConfigs(Configuration conf) {
|
||||
String trackerImpl = conf.get(TRACKER_IMPL, conf.get(StoreFileTrackerFactory.TRACKER_IMPL,
|
||||
|
|
|
@ -0,0 +1,40 @@
|
|||
/**
|
||||
* Licensed to the Apache Software Foundation (ASF) under one
|
||||
* or more contributor license agreements. See the NOTICE file
|
||||
* distributed with this work for additional information
|
||||
* regarding copyright ownership. The ASF licenses this file
|
||||
* to you under the Apache License, Version 2.0 (the
|
||||
* "License"); you may not use this file except in compliance
|
||||
* with the License. You may obtain a copy of the License at
|
||||
*
|
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
*
|
||||
* Unless required by applicable law or agreed to in writing, software
|
||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.hadoop.hbase.master;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.util.Collections;
|
||||
import java.util.Set;
|
||||
import org.apache.hadoop.hbase.ServerName;
|
||||
|
||||
public class DummyRegionServerList implements RegionServerList {
|
||||
|
||||
@Override
|
||||
public void started(ServerName sn) {
|
||||
}
|
||||
|
||||
@Override
|
||||
public void expired(ServerName sn) {
|
||||
}
|
||||
|
||||
@Override
|
||||
public Set<ServerName> getAll() throws IOException {
|
||||
return Collections.emptySet();
|
||||
}
|
||||
|
||||
}
|
|
@ -59,7 +59,7 @@ public class TestClockSkewDetection {
|
|||
when(conn.getRpcControllerFactory()).thenReturn(mock(RpcControllerFactory.class));
|
||||
return conn;
|
||||
}
|
||||
});
|
||||
}, new DummyRegionServerList());
|
||||
|
||||
LOG.debug("regionServerStartup 1");
|
||||
InetAddress ia1 = InetAddress.getLocalHost();
|
||||
|
|
|
@ -219,8 +219,9 @@ public class TestMasterNoCluster {
|
|||
protected void initClusterSchemaService() throws IOException, InterruptedException {}
|
||||
|
||||
@Override
|
||||
protected ServerManager createServerManager(MasterServices master) throws IOException {
|
||||
ServerManager sm = super.createServerManager(master);
|
||||
protected ServerManager createServerManager(MasterServices master, RegionServerList storage)
|
||||
throws IOException {
|
||||
ServerManager sm = super.createServerManager(master, storage);
|
||||
// Spy on the created servermanager
|
||||
ServerManager spy = Mockito.spy(sm);
|
||||
return spy;
|
||||
|
|
|
@ -0,0 +1,104 @@
|
|||
/**
|
||||
* Licensed to the Apache Software Foundation (ASF) under one
|
||||
* or more contributor license agreements. See the NOTICE file
|
||||
* distributed with this work for additional information
|
||||
* regarding copyright ownership. The ASF licenses this file
|
||||
* to you under the Apache License, Version 2.0 (the
|
||||
* "License"); you may not use this file except in compliance
|
||||
* with the License. You may obtain a copy of the License at
|
||||
*
|
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
*
|
||||
* Unless required by applicable law or agreed to in writing, software
|
||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.hadoop.hbase.master;
|
||||
|
||||
import static org.junit.Assert.assertArrayEquals;
|
||||
|
||||
import java.io.IOException;
|
||||
import org.apache.hadoop.fs.Path;
|
||||
import org.apache.hadoop.hbase.HBaseClassTestRule;
|
||||
import org.apache.hadoop.hbase.HBaseTestingUtility;
|
||||
import org.apache.hadoop.hbase.HConstants;
|
||||
import org.apache.hadoop.hbase.TableName;
|
||||
import org.apache.hadoop.hbase.client.Get;
|
||||
import org.apache.hadoop.hbase.client.Put;
|
||||
import org.apache.hadoop.hbase.client.Table;
|
||||
import org.apache.hadoop.hbase.testclassification.MasterTests;
|
||||
import org.apache.hadoop.hbase.testclassification.MediumTests;
|
||||
import org.apache.hadoop.hbase.util.Bytes;
|
||||
import org.apache.hadoop.hbase.util.CommonFSUtils;
|
||||
import org.junit.AfterClass;
|
||||
import org.junit.BeforeClass;
|
||||
import org.junit.ClassRule;
|
||||
import org.junit.Test;
|
||||
import org.junit.experimental.categories.Category;
|
||||
|
||||
/**
|
||||
* Simulate the scenario described in HBASE-26245, where we clean the WAL directory and try to start
|
||||
* the cluster.
|
||||
*/
|
||||
@Category({ MasterTests.class, MediumTests.class })
|
||||
public class TestRestartWithEmptyWALDirectory {
|
||||
|
||||
@ClassRule
|
||||
public static final HBaseClassTestRule CLASS_RULE =
|
||||
HBaseClassTestRule.forClass(TestRestartWithEmptyWALDirectory.class);
|
||||
|
||||
private static final HBaseTestingUtility UTIL = new HBaseTestingUtility();
|
||||
|
||||
private static TableName NAME = TableName.valueOf("test");
|
||||
|
||||
private static byte[] FAMILY = Bytes.toBytes("family");
|
||||
|
||||
private static byte[] QUALIFIER = Bytes.toBytes("qualifier");
|
||||
|
||||
@BeforeClass
|
||||
public static void setUp() throws Exception {
|
||||
// in the test we shutdown the only master and after restarting its port will be changed, so the
|
||||
// default rpc region server can not work
|
||||
UTIL.getConfiguration().set(HConstants.CLIENT_CONNECTION_REGISTRY_IMPL_CONF_KEY,
|
||||
HConstants.ZK_CONNECTION_REGISTRY_CLASS);
|
||||
UTIL.startMiniCluster(1);
|
||||
UTIL.createTable(NAME, FAMILY).close();
|
||||
UTIL.waitTableAvailable(NAME);
|
||||
}
|
||||
|
||||
@AfterClass
|
||||
public static void tearDown() throws IOException {
|
||||
UTIL.shutdownMiniCluster();
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testRestart() throws IOException, InterruptedException {
|
||||
byte[] row = Bytes.toBytes(0);
|
||||
try (Table table = UTIL.getConnection().getTable(NAME)) {
|
||||
table.put(new Put(row).addColumn(FAMILY, QUALIFIER, row));
|
||||
}
|
||||
// flush all in memory data
|
||||
UTIL.flush(TableName.META_TABLE_NAME);
|
||||
UTIL.flush(NAME);
|
||||
|
||||
// stop master first, so when stopping region server, we will not schedule a SCP.
|
||||
UTIL.getMiniHBaseCluster().stopMaster(0).join();
|
||||
UTIL.getMiniHBaseCluster().stopRegionServer(0).join();
|
||||
|
||||
// let's cleanup the WAL directory
|
||||
UTIL.getTestFileSystem().delete(new Path(CommonFSUtils.getWALRootDir(UTIL.getConfiguration()),
|
||||
HConstants.HREGION_LOGDIR_NAME), true);
|
||||
|
||||
// restart the cluster
|
||||
UTIL.getMiniHBaseCluster().startMaster();
|
||||
UTIL.getMiniHBaseCluster().startRegionServer();
|
||||
UTIL.waitTableAvailable(NAME);
|
||||
|
||||
// the start up should succeed and the data should be persist
|
||||
try (Table table = UTIL.getConnection().getTable(NAME)) {
|
||||
assertArrayEquals(row, table.get(new Get(row)).getValue(FAMILY, QUALIFIER));
|
||||
}
|
||||
}
|
||||
}
|
|
@ -39,6 +39,7 @@ import org.apache.hadoop.hbase.client.RegionInfoBuilder;
|
|||
import org.apache.hadoop.hbase.client.TableDescriptor;
|
||||
import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
|
||||
import org.apache.hadoop.hbase.client.TableState;
|
||||
import org.apache.hadoop.hbase.master.DummyRegionServerList;
|
||||
import org.apache.hadoop.hbase.master.LoadBalancer;
|
||||
import org.apache.hadoop.hbase.master.MasterFileSystem;
|
||||
import org.apache.hadoop.hbase.master.MasterServices;
|
||||
|
@ -117,7 +118,7 @@ public class MockMasterServices extends MockNoopMasterServices {
|
|||
this.assignmentManager =
|
||||
new AssignmentManager(this, masterRegion, new MockRegionStateStore(this, masterRegion));
|
||||
this.balancer = LoadBalancerFactory.getLoadBalancer(conf);
|
||||
this.serverManager = new ServerManager(this);
|
||||
this.serverManager = new ServerManager(this, new DummyRegionServerList());
|
||||
this.tableStateManager = Mockito.mock(TableStateManager.class);
|
||||
Mockito.when(this.tableStateManager.getTableState(Mockito.any())).
|
||||
thenReturn(new TableState(TableName.valueOf("AnyTableNameSetInMockMasterServcies"),
|
||||
|
|
|
@ -36,6 +36,7 @@ import org.apache.hadoop.hbase.client.RegionInfo;
|
|||
import org.apache.hadoop.hbase.master.HMaster;
|
||||
import org.apache.hadoop.hbase.master.MasterServices;
|
||||
import org.apache.hadoop.hbase.master.RegionPlan;
|
||||
import org.apache.hadoop.hbase.master.RegionServerList;
|
||||
import org.apache.hadoop.hbase.master.ServerManager;
|
||||
import org.apache.hadoop.hbase.master.region.MasterRegion;
|
||||
import org.apache.hadoop.hbase.regionserver.HRegionServer;
|
||||
|
@ -69,8 +70,8 @@ public class TestRegionAssignedToMultipleRegionServers {
|
|||
|
||||
private static final class ServerManagerForTest extends ServerManager {
|
||||
|
||||
public ServerManagerForTest(MasterServices master) {
|
||||
super(master);
|
||||
public ServerManagerForTest(MasterServices master, RegionServerList storage) {
|
||||
super(master, storage);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -121,9 +122,10 @@ public class TestRegionAssignedToMultipleRegionServers {
|
|||
}
|
||||
|
||||
@Override
|
||||
protected ServerManager createServerManager(MasterServices master) throws IOException {
|
||||
protected ServerManager createServerManager(MasterServices master,
|
||||
RegionServerList storage) throws IOException {
|
||||
setupClusterConnection();
|
||||
return new ServerManagerForTest(master);
|
||||
return new ServerManagerForTest(master, storage);
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
@ -39,6 +39,7 @@ import org.apache.hadoop.hbase.client.RegionInfo;
|
|||
import org.apache.hadoop.hbase.master.HMaster;
|
||||
import org.apache.hadoop.hbase.master.MasterServices;
|
||||
import org.apache.hadoop.hbase.master.RegionPlan;
|
||||
import org.apache.hadoop.hbase.master.RegionServerList;
|
||||
import org.apache.hadoop.hbase.master.ServerManager;
|
||||
import org.apache.hadoop.hbase.master.region.MasterRegion;
|
||||
import org.apache.hadoop.hbase.regionserver.HRegionServer;
|
||||
|
@ -71,8 +72,8 @@ public class TestReportRegionStateTransitionFromDeadServer {
|
|||
|
||||
private static final class ServerManagerForTest extends ServerManager {
|
||||
|
||||
public ServerManagerForTest(MasterServices master) {
|
||||
super(master);
|
||||
public ServerManagerForTest(MasterServices master, RegionServerList storage) {
|
||||
super(master, storage);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -128,9 +129,10 @@ public class TestReportRegionStateTransitionFromDeadServer {
|
|||
}
|
||||
|
||||
@Override
|
||||
protected ServerManager createServerManager(MasterServices master) throws IOException {
|
||||
protected ServerManager createServerManager(MasterServices master,
|
||||
RegionServerList storage) throws IOException {
|
||||
setupClusterConnection();
|
||||
return new ServerManagerForTest(master);
|
||||
return new ServerManagerForTest(master, storage);
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
@ -36,6 +36,7 @@ import org.apache.hadoop.hbase.client.RegionInfo;
|
|||
import org.apache.hadoop.hbase.master.HMaster;
|
||||
import org.apache.hadoop.hbase.master.MasterServices;
|
||||
import org.apache.hadoop.hbase.master.RegionPlan;
|
||||
import org.apache.hadoop.hbase.master.RegionServerList;
|
||||
import org.apache.hadoop.hbase.master.ServerManager;
|
||||
import org.apache.hadoop.hbase.master.procedure.ServerCrashProcedure;
|
||||
import org.apache.hadoop.hbase.master.region.MasterRegion;
|
||||
|
@ -70,8 +71,8 @@ public class TestSCPGetRegionsRace {
|
|||
|
||||
private static final class ServerManagerForTest extends ServerManager {
|
||||
|
||||
public ServerManagerForTest(MasterServices master) {
|
||||
super(master);
|
||||
public ServerManagerForTest(MasterServices master, RegionServerList storage) {
|
||||
super(master, storage);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -141,9 +142,10 @@ public class TestSCPGetRegionsRace {
|
|||
}
|
||||
|
||||
@Override
|
||||
protected ServerManager createServerManager(MasterServices master) throws IOException {
|
||||
protected ServerManager createServerManager(MasterServices master,
|
||||
RegionServerList storage) throws IOException {
|
||||
setupClusterConnection();
|
||||
return new ServerManagerForTest(master);
|
||||
return new ServerManagerForTest(master, storage);
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
@ -36,6 +36,7 @@ import org.apache.hadoop.hbase.client.RegionInfo;
|
|||
import org.apache.hadoop.hbase.master.HMaster;
|
||||
import org.apache.hadoop.hbase.master.MasterServices;
|
||||
import org.apache.hadoop.hbase.master.RegionPlan;
|
||||
import org.apache.hadoop.hbase.master.RegionServerList;
|
||||
import org.apache.hadoop.hbase.master.RegionState;
|
||||
import org.apache.hadoop.hbase.master.ServerManager;
|
||||
import org.apache.hadoop.hbase.master.region.MasterRegion;
|
||||
|
@ -165,8 +166,8 @@ public class TestWakeUpUnexpectedProcedure {
|
|||
|
||||
private static final class SMForTest extends ServerManager {
|
||||
|
||||
public SMForTest(MasterServices master) {
|
||||
super(master);
|
||||
public SMForTest(MasterServices master, RegionServerList storage) {
|
||||
super(master, storage);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -209,9 +210,10 @@ public class TestWakeUpUnexpectedProcedure {
|
|||
}
|
||||
|
||||
@Override
|
||||
protected ServerManager createServerManager(MasterServices master) throws IOException {
|
||||
protected ServerManager createServerManager(MasterServices master,
|
||||
RegionServerList storage) throws IOException {
|
||||
setupClusterConnection();
|
||||
return new SMForTest(master);
|
||||
return new SMForTest(master, storage);
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
@ -28,6 +28,7 @@ import org.apache.hadoop.hbase.TableName;
|
|||
import org.apache.hadoop.hbase.client.Table;
|
||||
import org.apache.hadoop.hbase.master.HMaster;
|
||||
import org.apache.hadoop.hbase.master.MasterServices;
|
||||
import org.apache.hadoop.hbase.master.RegionServerList;
|
||||
import org.apache.hadoop.hbase.master.ServerManager;
|
||||
import org.apache.hadoop.hbase.master.procedure.ServerCrashProcedure;
|
||||
import org.apache.hadoop.hbase.master.replication.ClaimReplicationQueuesProcedure;
|
||||
|
@ -67,8 +68,8 @@ public class TestClaimReplicationQueue extends TestReplicationBase {
|
|||
|
||||
public static final class ServerManagerForTest extends ServerManager {
|
||||
|
||||
public ServerManagerForTest(MasterServices master) {
|
||||
super(master);
|
||||
public ServerManagerForTest(MasterServices master, RegionServerList storage) {
|
||||
super(master, storage);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -92,9 +93,10 @@ public class TestClaimReplicationQueue extends TestReplicationBase {
|
|||
}
|
||||
|
||||
@Override
|
||||
protected ServerManager createServerManager(MasterServices master) throws IOException {
|
||||
protected ServerManager createServerManager(MasterServices master,
|
||||
RegionServerList storage) throws IOException {
|
||||
setupClusterConnection();
|
||||
return new ServerManagerForTest(master);
|
||||
return new ServerManagerForTest(master, storage);
|
||||
}
|
||||
}
|
||||
|
||||
|
|
Loading…
Reference in New Issue