Skip to content

Commit 34a166f

Browse files
committed
HBASE-27514 Move some persistent states from zookeeper to master region
1 parent 222ec68 commit 34a166f

25 files changed

+987
-508
lines changed

hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ZNodePaths.java

Lines changed: 21 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -69,10 +69,25 @@ public class ZNodePaths {
6969
// znode used for log splitting work assignment
7070
public final String splitLogZNode;
7171
// znode containing the state of the load balancer
72+
/**
73+
* @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
74+
* state.
75+
*/
76+
@Deprecated
7277
public final String balancerZNode;
7378
// znode containing the state of region normalizer
79+
/**
80+
* @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
81+
* state.
82+
*/
83+
@Deprecated
7484
public final String regionNormalizerZNode;
7585
// znode containing the state of all switches, currently there are split and merge child node.
86+
/**
87+
* @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
88+
* state.
89+
*/
90+
@Deprecated
7691
public final String switchZNode;
7792
// znode of indicating master maintenance mode
7893
public final String masterMaintZNode;
@@ -86,7 +101,12 @@ public class ZNodePaths {
86101
// znode containing queues of hfile references to be replicated
87102
public final String hfileRefsZNode;
88103
// znode containing the state of the snapshot auto-cleanup
89-
final String snapshotCleanupZNode;
104+
/**
105+
* @deprecated Since 2.6.0, will be removed in 4.0.0. We use master local region to store this
106+
* state.
107+
*/
108+
@Deprecated
109+
public final String snapshotCleanupZNode;
90110

91111
public ZNodePaths(Configuration conf) {
92112
baseZNode = conf.get(ZOOKEEPER_ZNODE_PARENT, DEFAULT_ZOOKEEPER_ZNODE_PARENT);
Lines changed: 63 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,63 @@
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.master;
19+
20+
import java.io.IOException;
21+
import org.apache.hadoop.hbase.exceptions.DeserializationException;
22+
import org.apache.hadoop.hbase.master.region.MasterRegion;
23+
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
24+
import org.apache.yetus.audience.InterfaceAudience;
25+
import org.apache.zookeeper.KeeperException;
26+
27+
/**
28+
* Store a boolean state.
29+
*/
30+
@InterfaceAudience.Private
31+
public abstract class BooleanStateStore extends MasterStateStore {
32+
33+
private volatile boolean on;
34+
35+
protected BooleanStateStore(MasterRegion masterRegion, String stateName, ZKWatcher watcher,
36+
String zkPath) throws IOException, KeeperException, DeserializationException {
37+
super(masterRegion, stateName, watcher, zkPath);
38+
byte[] state = getState();
39+
this.on = state == null || parseFrom(state);
40+
}
41+
42+
/**
43+
* Returns true if the flag is on, otherwise false
44+
*/
45+
public boolean get() {
46+
return on;
47+
}
48+
49+
/**
50+
* Set the flag on/off.
51+
* @param on true if the flag should be on, false otherwise
52+
* @throws IOException if the operation fails
53+
*/
54+
public synchronized void set(boolean on) throws IOException {
55+
byte[] state = toByteArray(on);
56+
setState(state);
57+
this.on = on;
58+
}
59+
60+
protected abstract byte[] toByteArray(boolean on);
61+
62+
protected abstract boolean parseFrom(byte[] bytes) throws DeserializationException;
63+
}

hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java

Lines changed: 34 additions & 43 deletions
Original file line numberDiff line numberDiff line change
@@ -109,6 +109,7 @@
109109
import org.apache.hadoop.hbase.client.TableState;
110110
import org.apache.hadoop.hbase.conf.ConfigurationManager;
111111
import org.apache.hadoop.hbase.coprocessor.CoprocessorHost;
112+
import org.apache.hadoop.hbase.exceptions.DeserializationException;
112113
import org.apache.hadoop.hbase.exceptions.MasterStoppedException;
113114
import org.apache.hadoop.hbase.executor.ExecutorType;
114115
import org.apache.hadoop.hbase.favored.FavoredNodesManager;
@@ -129,6 +130,7 @@
129130
import org.apache.hadoop.hbase.master.balancer.BaseLoadBalancer;
130131
import org.apache.hadoop.hbase.master.balancer.ClusterStatusChore;
131132
import org.apache.hadoop.hbase.master.balancer.LoadBalancerFactory;
133+
import org.apache.hadoop.hbase.master.balancer.LoadBalancerStateStore;
132134
import org.apache.hadoop.hbase.master.balancer.MaintenanceLoadBalancer;
133135
import org.apache.hadoop.hbase.master.cleaner.DirScanPool;
134136
import org.apache.hadoop.hbase.master.cleaner.HFileCleaner;
@@ -145,6 +147,7 @@
145147
import org.apache.hadoop.hbase.master.migrate.RollingUpgradeChore;
146148
import org.apache.hadoop.hbase.master.normalizer.RegionNormalizerFactory;
147149
import org.apache.hadoop.hbase.master.normalizer.RegionNormalizerManager;
150+
import org.apache.hadoop.hbase.master.normalizer.RegionNormalizerStateStore;
148151
import org.apache.hadoop.hbase.master.procedure.CreateTableProcedure;
149152
import org.apache.hadoop.hbase.master.procedure.DeleteNamespaceProcedure;
150153
import org.apache.hadoop.hbase.master.procedure.DeleteTableProcedure;
@@ -174,6 +177,7 @@
174177
import org.apache.hadoop.hbase.master.replication.TransitPeerSyncReplicationStateProcedure;
175178
import org.apache.hadoop.hbase.master.replication.UpdatePeerConfigProcedure;
176179
import org.apache.hadoop.hbase.master.slowlog.SlowLogMasterService;
180+
import org.apache.hadoop.hbase.master.snapshot.SnapshotCleanupStateStore;
177181
import org.apache.hadoop.hbase.master.snapshot.SnapshotManager;
178182
import org.apache.hadoop.hbase.master.waleventtracker.WALEventTrackerTableCreator;
179183
import org.apache.hadoop.hbase.master.zksyncer.MasterAddressSyncer;
@@ -246,11 +250,8 @@
246250
import org.apache.hadoop.hbase.util.TableDescriptorChecker;
247251
import org.apache.hadoop.hbase.util.Threads;
248252
import org.apache.hadoop.hbase.util.VersionInfo;
249-
import org.apache.hadoop.hbase.zookeeper.LoadBalancerTracker;
250253
import org.apache.hadoop.hbase.zookeeper.MasterAddressTracker;
251254
import org.apache.hadoop.hbase.zookeeper.MetaTableLocator;
252-
import org.apache.hadoop.hbase.zookeeper.RegionNormalizerTracker;
253-
import org.apache.hadoop.hbase.zookeeper.SnapshotCleanupTracker;
254255
import org.apache.hadoop.hbase.zookeeper.ZKClusterId;
255256
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
256257
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
@@ -306,17 +307,17 @@ public class HMaster extends HBaseServerBase<MasterRpcServices> implements Maste
306307
// Draining region server tracker
307308
private DrainingServerTracker drainingServerTracker;
308309
// Tracker for load balancer state
309-
LoadBalancerTracker loadBalancerTracker;
310+
LoadBalancerStateStore loadBalancerStateStore;
310311
// Tracker for meta location, if any client ZK quorum specified
311312
private MetaLocationSyncer metaLocationSyncer;
312313
// Tracker for active master location, if any client ZK quorum specified
313314
@InterfaceAudience.Private
314315
MasterAddressSyncer masterAddressSyncer;
315316
// Tracker for auto snapshot cleanup state
316-
SnapshotCleanupTracker snapshotCleanupTracker;
317+
SnapshotCleanupStateStore snapshotCleanupStateStore;
317318

318319
// Tracker for split and merge state
319-
private SplitOrMergeTracker splitOrMergeTracker;
320+
private SplitOrMergeStateStore splitOrMergeStateStore;
320321

321322
private ClusterSchemaService clusterSchemaService;
322323

@@ -750,24 +751,22 @@ public MetricsMaster getMasterMetrics() {
750751
* should have already been initialized along with {@link ServerManager}.
751752
*/
752753
private void initializeZKBasedSystemTrackers()
753-
throws IOException, KeeperException, ReplicationException {
754+
throws IOException, KeeperException, ReplicationException, DeserializationException {
754755
if (maintenanceMode) {
755756
// in maintenance mode, always use MaintenanceLoadBalancer.
756757
conf.unset(LoadBalancer.HBASE_RSGROUP_LOADBALANCER_CLASS);
757758
conf.setClass(HConstants.HBASE_MASTER_LOADBALANCER_CLASS, MaintenanceLoadBalancer.class,
758759
LoadBalancer.class);
759760
}
760761
this.balancer = new RSGroupBasedLoadBalancer();
761-
this.loadBalancerTracker = new LoadBalancerTracker(zooKeeper, this);
762-
this.loadBalancerTracker.start();
762+
this.loadBalancerStateStore = new LoadBalancerStateStore(masterRegion, zooKeeper);
763763

764764
this.regionNormalizerManager =
765-
RegionNormalizerFactory.createNormalizerManager(conf, zooKeeper, this);
765+
RegionNormalizerFactory.createNormalizerManager(conf, masterRegion, zooKeeper, this);
766766
this.configurationManager.registerObserver(regionNormalizerManager);
767767
this.regionNormalizerManager.start();
768768

769-
this.splitOrMergeTracker = new SplitOrMergeTracker(zooKeeper, conf, this);
770-
this.splitOrMergeTracker.start();
769+
this.splitOrMergeStateStore = new SplitOrMergeStateStore(masterRegion, zooKeeper, conf);
771770

772771
// This is for backwards compatible. We do not need the CP for rs group now but if user want to
773772
// load it, we need to enable rs group.
@@ -787,8 +786,7 @@ private void initializeZKBasedSystemTrackers()
787786
this.drainingServerTracker = new DrainingServerTracker(zooKeeper, this, this.serverManager);
788787
this.drainingServerTracker.start();
789788

790-
this.snapshotCleanupTracker = new SnapshotCleanupTracker(zooKeeper, this);
791-
this.snapshotCleanupTracker.start();
789+
this.snapshotCleanupStateStore = new SnapshotCleanupStateStore(masterRegion, zooKeeper);
792790

793791
String clientQuorumServers = conf.get(HConstants.CLIENT_ZOOKEEPER_QUORUM);
794792
boolean clientZkObserverMode = conf.getBoolean(HConstants.CLIENT_ZOOKEEPER_OBSERVER_MODE,
@@ -910,8 +908,8 @@ private void tryMigrateMetaLocationsFromZooKeeper() throws IOException, KeeperEx
910908
* Notice that now we will not schedule a special procedure to make meta online(unless the first
911909
* time where meta has not been created yet), we will rely on SCP to bring meta online.
912910
*/
913-
private void finishActiveMasterInitialization(MonitoredTask status)
914-
throws IOException, InterruptedException, KeeperException, ReplicationException {
911+
private void finishActiveMasterInitialization(MonitoredTask status) throws IOException,
912+
InterruptedException, KeeperException, ReplicationException, DeserializationException {
915913
/*
916914
* We are active master now... go initialize components we need to run.
917915
*/
@@ -1640,7 +1638,7 @@ conf, getMasterFileSystem().getFileSystem(), new Path(archiveDir, path),
16401638
new ReplicationBarrierCleaner(conf, this, getConnection(), replicationPeerManager);
16411639
getChoreService().scheduleChore(replicationBarrierCleaner);
16421640

1643-
final boolean isSnapshotChoreEnabled = this.snapshotCleanupTracker.isSnapshotCleanupEnabled();
1641+
final boolean isSnapshotChoreEnabled = this.snapshotCleanupStateStore.get();
16441642
this.snapshotCleanerChore = new SnapshotCleanerChore(this, conf, getSnapshotManager());
16451643
if (isSnapshotChoreEnabled) {
16461644
getChoreService().scheduleChore(this.snapshotCleanerChore);
@@ -1762,7 +1760,7 @@ protected void startProcedureExecutor() throws IOException {
17621760
* Turn on/off Snapshot Cleanup Chore
17631761
* @param on indicates whether Snapshot Cleanup Chore is to be run
17641762
*/
1765-
void switchSnapshotCleanup(final boolean on, final boolean synchronous) {
1763+
void switchSnapshotCleanup(final boolean on, final boolean synchronous) throws IOException {
17661764
if (synchronous) {
17671765
synchronized (this.snapshotCleanerChore) {
17681766
switchSnapshotCleanup(on);
@@ -1772,16 +1770,12 @@ void switchSnapshotCleanup(final boolean on, final boolean synchronous) {
17721770
}
17731771
}
17741772

1775-
private void switchSnapshotCleanup(final boolean on) {
1776-
try {
1777-
snapshotCleanupTracker.setSnapshotCleanupEnabled(on);
1778-
if (on) {
1779-
getChoreService().scheduleChore(this.snapshotCleanerChore);
1780-
} else {
1781-
this.snapshotCleanerChore.cancel();
1782-
}
1783-
} catch (KeeperException e) {
1784-
LOG.error("Error updating snapshot cleanup mode to {}", on, e);
1773+
private void switchSnapshotCleanup(final boolean on) throws IOException {
1774+
snapshotCleanupStateStore.set(on);
1775+
if (on) {
1776+
getChoreService().scheduleChore(this.snapshotCleanerChore);
1777+
} else {
1778+
this.snapshotCleanerChore.cancel();
17851779
}
17861780
}
17871781

@@ -1955,9 +1949,7 @@ public BalanceResponse balance(BalanceRequest request) throws IOException {
19551949

19561950
BalanceResponse.Builder responseBuilder = BalanceResponse.newBuilder();
19571951

1958-
if (
1959-
loadBalancerTracker == null || !(loadBalancerTracker.isBalancerOn() || request.isDryRun())
1960-
) {
1952+
if (loadBalancerStateStore == null || !(loadBalancerStateStore.get() || request.isDryRun())) {
19611953
return responseBuilder.build();
19621954
}
19631955

@@ -2889,8 +2881,8 @@ public ClusterMetrics getClusterMetricsWithoutCoprocessor(EnumSet<Option> option
28892881
break;
28902882
}
28912883
case BALANCER_ON: {
2892-
if (loadBalancerTracker != null) {
2893-
builder.setBalancerOn(loadBalancerTracker.isBalancerOn());
2884+
if (loadBalancerStateStore != null) {
2885+
builder.setBalancerOn(loadBalancerStateStore.get());
28942886
}
28952887
break;
28962888
}
@@ -3727,33 +3719,32 @@ public void reportMobCompactionEnd(TableName tableName) throws IOException {
37273719
}
37283720

37293721
/**
3730-
* Queries the state of the {@link LoadBalancerTracker}. If the balancer is not initialized, false
3731-
* is returned.
3722+
* Queries the state of the {@link LoadBalancerStateStore}. If the balancer is not initialized,
3723+
* false is returned.
37323724
* @return The state of the load balancer, or false if the load balancer isn't defined.
37333725
*/
37343726
public boolean isBalancerOn() {
3735-
return !isInMaintenanceMode() && loadBalancerTracker != null
3736-
&& loadBalancerTracker.isBalancerOn();
3727+
return !isInMaintenanceMode() && loadBalancerStateStore != null && loadBalancerStateStore.get();
37373728
}
37383729

37393730
/**
3740-
* Queries the state of the {@link RegionNormalizerTracker}. If it's not initialized, false is
3731+
* Queries the state of the {@link RegionNormalizerStateStore}. If it's not initialized, false is
37413732
* returned.
37423733
*/
37433734
public boolean isNormalizerOn() {
37443735
return !isInMaintenanceMode() && getRegionNormalizerManager().isNormalizerOn();
37453736
}
37463737

37473738
/**
3748-
* Queries the state of the {@link SplitOrMergeTracker}. If it is not initialized, false is
3739+
* Queries the state of the {@link SplitOrMergeStateStore}. If it is not initialized, false is
37493740
* returned. If switchType is illegal, false will return.
37503741
* @param switchType see {@link org.apache.hadoop.hbase.client.MasterSwitchType}
37513742
* @return The state of the switch
37523743
*/
37533744
@Override
37543745
public boolean isSplitOrMergeEnabled(MasterSwitchType switchType) {
3755-
return !isInMaintenanceMode() && splitOrMergeTracker != null
3756-
&& splitOrMergeTracker.isSplitOrMergeEnabled(switchType);
3746+
return !isInMaintenanceMode() && splitOrMergeStateStore != null
3747+
&& splitOrMergeStateStore.isSplitOrMergeEnabled(switchType);
37573748
}
37583749

37593750
/**
@@ -3768,8 +3759,8 @@ public String getLoadBalancerClassName() {
37683759
LoadBalancerFactory.getDefaultLoadBalancerClass().getName());
37693760
}
37703761

3771-
public SplitOrMergeTracker getSplitOrMergeTracker() {
3772-
return splitOrMergeTracker;
3762+
public SplitOrMergeStateStore getSplitOrMergeStateStore() {
3763+
return splitOrMergeStateStore;
37733764
}
37743765

37753766
@Override

0 commit comments

Comments
 (0)