HBASE-16598 Enable zookeeper useMulti always and clean up in HBase code

This commit is contained in:
Jerry He 2016-09-17 16:51:26 -07:00
parent bb3d9ccd48
commit edc0ef3fe4
17 changed files with 50 additions and 1003 deletions

View File

@ -131,7 +131,6 @@ public class ReplicationPeersZKImpl extends ReplicationStateZKBase implements Re
List<ZKUtilOp> listOfOps = new ArrayList<ZKUtil.ZKUtilOp>();
ZKUtilOp op1 = ZKUtilOp.createAndFailSilent(getPeerNode(id),
ReplicationSerDeHelper.toByteArray(peerConfig));
// There is a race (if hbase.zookeeper.useMulti is false)
// b/w PeerWatcher and ReplicationZookeeper#add method to create the
// peer-state znode. This happens while adding a peer
// The peer state data is set as "ENABLED" by default.

View File

@ -23,7 +23,6 @@ import java.util.List;
import java.util.SortedSet;
import java.util.TreeSet;
import com.google.common.annotations.VisibleForTesting;
import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.hbase.classification.InterfaceAudience;
@ -31,8 +30,6 @@ import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hadoop.hbase.HConstants;
import org.apache.hadoop.hbase.exceptions.DeserializationException;
import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.Pair;
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
@ -67,8 +64,6 @@ public class ReplicationQueuesZKImpl extends ReplicationStateZKBase implements R
/** Znode containing all replication queues for this region server. */
private String myQueuesZnode;
/** Name of znode we use to lock during failover */
private final static String RS_LOCK_ZNODE = "lock";
private static final Log LOG = LogFactory.getLog(ReplicationQueuesZKImpl.class);
@ -189,42 +184,13 @@ public class ReplicationQueuesZKImpl extends ReplicationStateZKBase implements R
} catch (KeeperException e) {
this.abortable.abort("Failed to getUnClaimedQueueIds for RS" + regionserver, e);
}
if (queues != null) {
queues.remove(RS_LOCK_ZNODE);
}
return queues;
}
@Override
public Pair<String, SortedSet<String>> claimQueue(String regionserver, String queueId) {
if (conf.getBoolean(HConstants.ZOOKEEPER_USEMULTI, true)) {
LOG.info("Atomically moving " + regionserver + "/" + queueId + "'s WALs to my queue");
return moveQueueUsingMulti(regionserver, queueId);
} else {
LOG.info("Moving " + regionserver + "/" + queueId + "'s wals to my queue");
if (!lockOtherRS(regionserver)) {
LOG.info("Can not take the lock now");
return null;
}
Pair<String, SortedSet<String>> newQueues;
try {
newQueues = copyQueueFromLockedRS(regionserver, queueId);
removeQueueFromLockedRS(regionserver, queueId);
} finally {
unlockOtherRS(regionserver);
}
return newQueues;
}
}
private void removeQueueFromLockedRS(String znode, String peerId) {
String nodePath = ZKUtil.joinZNode(this.queuesZNode, znode);
String peerPath = ZKUtil.joinZNode(nodePath, peerId);
try {
ZKUtil.deleteNodeRecursively(this.zookeeper, peerPath);
} catch (KeeperException e) {
LOG.warn("Remove copied queue failed", e);
}
LOG.info("Atomically moving " + regionserver + "/" + queueId + "'s WALs to my queue");
return moveQueueUsingMulti(regionserver, queueId);
}
@Override
@ -278,58 +244,6 @@ public class ReplicationQueuesZKImpl extends ReplicationStateZKBase implements R
return listOfQueues == null ? new ArrayList<String>() : listOfQueues;
}
/**
* Try to set a lock in another region server's znode.
* @param znode the server names of the other server
* @return true if the lock was acquired, false in every other cases
*/
@VisibleForTesting
public boolean lockOtherRS(String znode) {
try {
String parent = ZKUtil.joinZNode(this.queuesZNode, znode);
if (parent.equals(this.myQueuesZnode)) {
LOG.warn("Won't lock because this is us, we're dead!");
return false;
}
String p = ZKUtil.joinZNode(parent, RS_LOCK_ZNODE);
ZKUtil.createAndWatch(this.zookeeper, p, lockToByteArray(this.myQueuesZnode));
} catch (KeeperException e) {
// This exception will pop up if the znode under which we're trying to
// create the lock is already deleted by another region server, meaning
// that the transfer already occurred.
// NoNode => transfer is done and znodes are already deleted
// NodeExists => lock znode already created by another RS
if (e instanceof KeeperException.NoNodeException
|| e instanceof KeeperException.NodeExistsException) {
LOG.info("Won't transfer the queue," + " another RS took care of it because of: "
+ e.getMessage());
} else {
LOG.info("Failed lock other rs", e);
}
return false;
}
return true;
}
public String getLockZNode(String znode) {
return this.queuesZNode + "/" + znode + "/" + RS_LOCK_ZNODE;
}
@VisibleForTesting
public boolean checkLockExists(String znode) throws KeeperException {
return ZKUtil.checkExists(zookeeper, getLockZNode(znode)) >= 0;
}
private void unlockOtherRS(String znode){
String parent = ZKUtil.joinZNode(this.queuesZNode, znode);
String p = ZKUtil.joinZNode(parent, RS_LOCK_ZNODE);
try {
ZKUtil.deleteNode(this.zookeeper, p);
} catch (KeeperException e) {
this.abortable.abort("Remove lock failed", e);
}
}
/**
* It "atomically" copies all the wals queues from another region server and returns them all
* sorted per peer cluster (appended with the dead server's znode).
@ -390,76 +304,6 @@ public class ReplicationQueuesZKImpl extends ReplicationStateZKBase implements R
return null;
}
/**
* This methods moves all the wals queues from another region server and returns them all sorted
* per peer cluster (appended with the dead server's znode)
* @param znode server names to copy
* @return all wals for the peer of that cluster, null if an error occurred
*/
private Pair<String, SortedSet<String>> copyQueueFromLockedRS(String znode, String peerId) {
// TODO this method isn't atomic enough, we could start copying and then
// TODO fail for some reason and we would end up with znodes we don't want.
try {
String nodePath = ZKUtil.joinZNode(this.queuesZNode, znode);
ReplicationQueueInfo replicationQueueInfo = new ReplicationQueueInfo(peerId);
String clusterPath = ZKUtil.joinZNode(nodePath, peerId);
if (!peerExists(replicationQueueInfo.getPeerId())) {
LOG.warn("Peer " + peerId + " didn't exist, skipping the replay");
// Protection against moving orphaned queues
return null;
}
// We add the name of the recovered RS to the new znode, we can even
// do that for queues that were recovered 10 times giving a znode like
// number-startcode-number-otherstartcode-number-anotherstartcode-etc
String newCluster = peerId + "-" + znode;
String newClusterZnode = ZKUtil.joinZNode(this.myQueuesZnode, newCluster);
List<String> wals = ZKUtil.listChildrenNoWatch(this.zookeeper, clusterPath);
// That region server didn't have anything to replicate for this cluster
if (wals == null || wals.size() == 0) {
return null;
}
ZKUtil.createNodeIfNotExistsAndWatch(this.zookeeper, newClusterZnode,
HConstants.EMPTY_BYTE_ARRAY);
SortedSet<String> logQueue = new TreeSet<>();
for (String wal : wals) {
String z = ZKUtil.joinZNode(clusterPath, wal);
byte[] positionBytes = ZKUtil.getData(this.zookeeper, z);
long position = 0;
try {
position = ZKUtil.parseWALPositionFrom(positionBytes);
} catch (DeserializationException e) {
LOG.warn("Failed parse of wal position from the following znode: " + z
+ ", Exception: " + e);
}
LOG.debug("Creating " + wal + " with data " + position);
String child = ZKUtil.joinZNode(newClusterZnode, wal);
// Position doesn't actually change, we are just deserializing it for
// logging, so just use the already serialized version
ZKUtil.createNodeIfNotExistsAndWatch(this.zookeeper, child, positionBytes);
logQueue.add(wal);
}
return new Pair<>(newCluster, logQueue);
} catch (KeeperException e) {
LOG.warn("Got exception in copyQueueFromLockedRS: ", e);
} catch (InterruptedException e) {
LOG.warn(e);
Thread.currentThread().interrupt();
}
return null;
}
/**
* @param lockOwner
* @return Serialized protobuf of <code>lockOwner</code> with pb magic prefix prepended suitable
* for use as content of an replication lock during region server fail over.
*/
static byte[] lockToByteArray(final String lockOwner) {
byte[] bytes =
ZooKeeperProtos.ReplicationLock.newBuilder().setLockOwner(lockOwner).build().toByteArray();
return ProtobufUtil.prependPBMagic(bytes);
}
@Override
public void addHFileRefs(String peerId, List<String> files) throws ReplicationException {
String peerZnode = ZKUtil.joinZNode(this.hfileRefsZNode, peerId);

View File

@ -1285,9 +1285,6 @@ public class ZKUtil {
* Sets no watches. Throws all exceptions besides dealing with deletion of
* children.
*
* If hbase.zookeeper.useMulti is true, use ZooKeeper's multi-update functionality.
* Otherwise, run the list of operations sequentially.
*
* @throws KeeperException
*/
public static void deleteChildrenRecursively(ZooKeeperWatcher zkw, String node)
@ -1304,13 +1301,9 @@ public class ZKUtil {
* Sets no watches. Throws all exceptions besides dealing with deletion of
* children.
* <p>
* If hbase.zookeeper.useMulti is true, use ZooKeeper's multi-update
* functionality. Otherwise, run the list of operations sequentially.
* <p>
* If all of the following are true:
* If the following is true:
* <ul>
* <li>runSequentialOnMultiFailure is true
* <li>hbase.zookeeper.useMulti is true
* </ul>
* on calling multi, we get a ZooKeeper exception that can be handled by a
* sequential call(*), we retry the operations one-by-one (sequentially).
@ -1359,13 +1352,9 @@ public class ZKUtil {
* Sets no watches. Throws all exceptions besides dealing with deletion of
* children.
* <p>
* If hbase.zookeeper.useMulti is true, use ZooKeeper's multi-update
* functionality. Otherwise, run the list of operations sequentially.
* <p>
* If all of the following are true:
* If the following is true:
* <ul>
* <li>runSequentialOnMultiFailure is true
* <li>hbase.zookeeper.useMulti is true
* </ul>
* on calling multi, we get a ZooKeeper exception that can be handled by a
* sequential call(*), we retry the operations one-by-one (sequentially).
@ -1636,12 +1625,10 @@ public class ZKUtil {
}
/**
* If hbase.zookeeper.useMulti is true, use ZooKeeper's multi-update functionality.
* Otherwise, run the list of operations sequentially.
* Use ZooKeeper's multi-update functionality.
*
* If all of the following are true:
* - runSequentialOnMultiFailure is true
* - hbase.zookeeper.useMulti is true
* - on calling multi, we get a ZooKeeper exception that can be handled by a sequential call(*)
* Then:
* - we retry the operations one-by-one (sequentially)
@ -1658,42 +1645,38 @@ public class ZKUtil {
*/
public static void multiOrSequential(ZooKeeperWatcher zkw, List<ZKUtilOp> ops,
boolean runSequentialOnMultiFailure) throws KeeperException {
if (ops == null) return;
boolean useMulti = zkw.getConfiguration().getBoolean(HConstants.ZOOKEEPER_USEMULTI, false);
if (useMulti) {
List<Op> zkOps = new LinkedList<Op>();
for (ZKUtilOp op : ops) {
zkOps.add(toZooKeeperOp(zkw, op));
}
try {
zkw.getRecoverableZooKeeper().multi(zkOps);
} catch (KeeperException ke) {
switch (ke.code()) {
case NODEEXISTS:
case NONODE:
case BADVERSION:
case NOAUTH:
// if we get an exception that could be solved by running sequentially
// (and the client asked us to), then break out and run sequentially
if (runSequentialOnMultiFailure) {
LOG.info("On call to ZK.multi, received exception: " + ke.toString() + "."
+ " Attempting to run operations sequentially because"
+ " runSequentialOnMultiFailure is: " + runSequentialOnMultiFailure + ".");
processSequentially(zkw, ops);
break;
}
default:
throw ke;
}
} catch (InterruptedException ie) {
zkw.interruptedException(ie);
}
} else {
// run sequentially
processSequentially(zkw, ops);
if (zkw.getConfiguration().get("hbase.zookeeper.useMulti") != null) {
LOG.warn("hbase.zookeeper.useMulti is deprecated. Default to true always.");
}
if (ops == null) return;
List<Op> zkOps = new LinkedList<Op>();
for (ZKUtilOp op : ops) {
zkOps.add(toZooKeeperOp(zkw, op));
}
try {
zkw.getRecoverableZooKeeper().multi(zkOps);
} catch (KeeperException ke) {
switch (ke.code()) {
case NODEEXISTS:
case NONODE:
case BADVERSION:
case NOAUTH:
// if we get an exception that could be solved by running sequentially
// (and the client asked us to), then break out and run sequentially
if (runSequentialOnMultiFailure) {
LOG.info("On call to ZK.multi, received exception: " + ke.toString() + "."
+ " Attempting to run operations sequentially because"
+ " runSequentialOnMultiFailure is: " + runSequentialOnMultiFailure + ".");
processSequentially(zkw, ops);
break;
}
default:
throw ke;
}
} catch (InterruptedException ie) {
zkw.interruptedException(ie);
}
}
private static void processSequentially(ZooKeeperWatcher zkw, List<ZKUtilOp> ops)

View File

@ -229,9 +229,6 @@ public final class HConstants {
/** Default value for ZooKeeper session timeout */
public static final int DEFAULT_ZK_SESSION_TIMEOUT = 180 * 1000;
/** Configuration key for whether to use ZK.multi */
public static final String ZOOKEEPER_USEMULTI = "hbase.zookeeper.useMulti";
/** Parameter name for port region server listens on. */
public static final String REGIONSERVER_PORT = "hbase.regionserver.port";

View File

@ -398,16 +398,6 @@ possible configurations would overwhelm and obscure the important.
for more information.</description>
</property>
<!-- End of properties used to generate ZooKeeper host:port quorum list. -->
<property>
<name>hbase.zookeeper.useMulti</name>
<value>true</value>
<description>Instructs HBase to make use of ZooKeeper's multi-update functionality.
This allows certain ZooKeeper operations to complete more quickly and prevents some issues
with rare Replication failure scenarios (see the release note of HBASE-2611 for an example).
IMPORTANT: only set this to true if all ZooKeeper servers in the cluster are on version 3.4+
and will not be downgraded. ZooKeeper versions before 3.4 do not support multi-update and
will not fail gracefully if multi-update is invoked (see ZOOKEEPER-1495).</description>
</property>
<!--
Beginning of properties that are directly mapped from ZooKeeper's zoo.cfg.

View File

@ -7703,533 +7703,6 @@ public final class ZooKeeperProtos {
// @@protoc_insertion_point(class_scope:hbase.pb.ReplicationHLogPosition)
}
public interface ReplicationLockOrBuilder
extends com.google.protobuf.MessageOrBuilder {
// required string lock_owner = 1;
/**
* <code>required string lock_owner = 1;</code>
*/
boolean hasLockOwner();
/**
* <code>required string lock_owner = 1;</code>
*/
java.lang.String getLockOwner();
/**
* <code>required string lock_owner = 1;</code>
*/
com.google.protobuf.ByteString
getLockOwnerBytes();
}
/**
* Protobuf type {@code hbase.pb.ReplicationLock}
*
* <pre>
**
* Used by replication. Used to lock a region server during failover.
* </pre>
*/
public static final class ReplicationLock extends
com.google.protobuf.GeneratedMessage
implements ReplicationLockOrBuilder {
// Use ReplicationLock.newBuilder() to construct.
private ReplicationLock(com.google.protobuf.GeneratedMessage.Builder<?> builder) {
super(builder);
this.unknownFields = builder.getUnknownFields();
}
private ReplicationLock(boolean noInit) { this.unknownFields = com.google.protobuf.UnknownFieldSet.getDefaultInstance(); }
private static final ReplicationLock defaultInstance;
public static ReplicationLock getDefaultInstance() {
return defaultInstance;
}
public ReplicationLock getDefaultInstanceForType() {
return defaultInstance;
}
private final com.google.protobuf.UnknownFieldSet unknownFields;
@java.lang.Override
public final com.google.protobuf.UnknownFieldSet
getUnknownFields() {
return this.unknownFields;
}
private ReplicationLock(
com.google.protobuf.CodedInputStream input,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws com.google.protobuf.InvalidProtocolBufferException {
initFields();
int mutable_bitField0_ = 0;
com.google.protobuf.UnknownFieldSet.Builder unknownFields =
com.google.protobuf.UnknownFieldSet.newBuilder();
try {
boolean done = false;
while (!done) {
int tag = input.readTag();
switch (tag) {
case 0:
done = true;
break;
default: {
if (!parseUnknownField(input, unknownFields,
extensionRegistry, tag)) {
done = true;
}
break;
}
case 10: {
bitField0_ |= 0x00000001;
lockOwner_ = input.readBytes();
break;
}
}
}
} catch (com.google.protobuf.InvalidProtocolBufferException e) {
throw e.setUnfinishedMessage(this);
} catch (java.io.IOException e) {
throw new com.google.protobuf.InvalidProtocolBufferException(
e.getMessage()).setUnfinishedMessage(this);
} finally {
this.unknownFields = unknownFields.build();
makeExtensionsImmutable();
}
}
public static final com.google.protobuf.Descriptors.Descriptor
getDescriptor() {
return org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.internal_static_hbase_pb_ReplicationLock_descriptor;
}
protected com.google.protobuf.GeneratedMessage.FieldAccessorTable
internalGetFieldAccessorTable() {
return org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.internal_static_hbase_pb_ReplicationLock_fieldAccessorTable
.ensureFieldAccessorsInitialized(
org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.class, org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.Builder.class);
}
public static com.google.protobuf.Parser<ReplicationLock> PARSER =
new com.google.protobuf.AbstractParser<ReplicationLock>() {
public ReplicationLock parsePartialFrom(
com.google.protobuf.CodedInputStream input,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws com.google.protobuf.InvalidProtocolBufferException {
return new ReplicationLock(input, extensionRegistry);
}
};
@java.lang.Override
public com.google.protobuf.Parser<ReplicationLock> getParserForType() {
return PARSER;
}
private int bitField0_;
// required string lock_owner = 1;
public static final int LOCK_OWNER_FIELD_NUMBER = 1;
private java.lang.Object lockOwner_;
/**
* <code>required string lock_owner = 1;</code>
*/
public boolean hasLockOwner() {
return ((bitField0_ & 0x00000001) == 0x00000001);
}
/**
* <code>required string lock_owner = 1;</code>
*/
public java.lang.String getLockOwner() {
java.lang.Object ref = lockOwner_;
if (ref instanceof java.lang.String) {
return (java.lang.String) ref;
} else {
com.google.protobuf.ByteString bs =
(com.google.protobuf.ByteString) ref;
java.lang.String s = bs.toStringUtf8();
if (bs.isValidUtf8()) {
lockOwner_ = s;
}
return s;
}
}
/**
* <code>required string lock_owner = 1;</code>
*/
public com.google.protobuf.ByteString
getLockOwnerBytes() {
java.lang.Object ref = lockOwner_;
if (ref instanceof java.lang.String) {
com.google.protobuf.ByteString b =
com.google.protobuf.ByteString.copyFromUtf8(
(java.lang.String) ref);
lockOwner_ = b;
return b;
} else {
return (com.google.protobuf.ByteString) ref;
}
}
private void initFields() {
lockOwner_ = "";
}
private byte memoizedIsInitialized = -1;
public final boolean isInitialized() {
byte isInitialized = memoizedIsInitialized;
if (isInitialized != -1) return isInitialized == 1;
if (!hasLockOwner()) {
memoizedIsInitialized = 0;
return false;
}
memoizedIsInitialized = 1;
return true;
}
public void writeTo(com.google.protobuf.CodedOutputStream output)
throws java.io.IOException {
getSerializedSize();
if (((bitField0_ & 0x00000001) == 0x00000001)) {
output.writeBytes(1, getLockOwnerBytes());
}
getUnknownFields().writeTo(output);
}
private int memoizedSerializedSize = -1;
public int getSerializedSize() {
int size = memoizedSerializedSize;
if (size != -1) return size;
size = 0;
if (((bitField0_ & 0x00000001) == 0x00000001)) {
size += com.google.protobuf.CodedOutputStream
.computeBytesSize(1, getLockOwnerBytes());
}
size += getUnknownFields().getSerializedSize();
memoizedSerializedSize = size;
return size;
}
private static final long serialVersionUID = 0L;
@java.lang.Override
protected java.lang.Object writeReplace()
throws java.io.ObjectStreamException {
return super.writeReplace();
}
@java.lang.Override
public boolean equals(final java.lang.Object obj) {
if (obj == this) {
return true;
}
if (!(obj instanceof org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock)) {
return super.equals(obj);
}
org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock other = (org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock) obj;
boolean result = true;
result = result && (hasLockOwner() == other.hasLockOwner());
if (hasLockOwner()) {
result = result && getLockOwner()
.equals(other.getLockOwner());
}
result = result &&
getUnknownFields().equals(other.getUnknownFields());
return result;
}
private int memoizedHashCode = 0;
@java.lang.Override
public int hashCode() {
if (memoizedHashCode != 0) {
return memoizedHashCode;
}
int hash = 41;
hash = (19 * hash) + getDescriptorForType().hashCode();
if (hasLockOwner()) {
hash = (37 * hash) + LOCK_OWNER_FIELD_NUMBER;
hash = (53 * hash) + getLockOwner().hashCode();
}
hash = (29 * hash) + getUnknownFields().hashCode();
memoizedHashCode = hash;
return hash;
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(
com.google.protobuf.ByteString data)
throws com.google.protobuf.InvalidProtocolBufferException {
return PARSER.parseFrom(data);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(
com.google.protobuf.ByteString data,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws com.google.protobuf.InvalidProtocolBufferException {
return PARSER.parseFrom(data, extensionRegistry);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(byte[] data)
throws com.google.protobuf.InvalidProtocolBufferException {
return PARSER.parseFrom(data);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(
byte[] data,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws com.google.protobuf.InvalidProtocolBufferException {
return PARSER.parseFrom(data, extensionRegistry);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(java.io.InputStream input)
throws java.io.IOException {
return PARSER.parseFrom(input);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(
java.io.InputStream input,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws java.io.IOException {
return PARSER.parseFrom(input, extensionRegistry);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseDelimitedFrom(java.io.InputStream input)
throws java.io.IOException {
return PARSER.parseDelimitedFrom(input);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseDelimitedFrom(
java.io.InputStream input,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws java.io.IOException {
return PARSER.parseDelimitedFrom(input, extensionRegistry);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(
com.google.protobuf.CodedInputStream input)
throws java.io.IOException {
return PARSER.parseFrom(input);
}
public static org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parseFrom(
com.google.protobuf.CodedInputStream input,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws java.io.IOException {
return PARSER.parseFrom(input, extensionRegistry);
}
public static Builder newBuilder() { return Builder.create(); }
public Builder newBuilderForType() { return newBuilder(); }
public static Builder newBuilder(org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock prototype) {
return newBuilder().mergeFrom(prototype);
}
public Builder toBuilder() { return newBuilder(this); }
@java.lang.Override
protected Builder newBuilderForType(
com.google.protobuf.GeneratedMessage.BuilderParent parent) {
Builder builder = new Builder(parent);
return builder;
}
/**
* Protobuf type {@code hbase.pb.ReplicationLock}
*
* <pre>
**
* Used by replication. Used to lock a region server during failover.
* </pre>
*/
public static final class Builder extends
com.google.protobuf.GeneratedMessage.Builder<Builder>
implements org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLockOrBuilder {
public static final com.google.protobuf.Descriptors.Descriptor
getDescriptor() {
return org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.internal_static_hbase_pb_ReplicationLock_descriptor;
}
protected com.google.protobuf.GeneratedMessage.FieldAccessorTable
internalGetFieldAccessorTable() {
return org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.internal_static_hbase_pb_ReplicationLock_fieldAccessorTable
.ensureFieldAccessorsInitialized(
org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.class, org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.Builder.class);
}
// Construct using org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.newBuilder()
private Builder() {
maybeForceBuilderInitialization();
}
private Builder(
com.google.protobuf.GeneratedMessage.BuilderParent parent) {
super(parent);
maybeForceBuilderInitialization();
}
private void maybeForceBuilderInitialization() {
if (com.google.protobuf.GeneratedMessage.alwaysUseFieldBuilders) {
}
}
private static Builder create() {
return new Builder();
}
public Builder clear() {
super.clear();
lockOwner_ = "";
bitField0_ = (bitField0_ & ~0x00000001);
return this;
}
public Builder clone() {
return create().mergeFrom(buildPartial());
}
public com.google.protobuf.Descriptors.Descriptor
getDescriptorForType() {
return org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.internal_static_hbase_pb_ReplicationLock_descriptor;
}
public org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock getDefaultInstanceForType() {
return org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.getDefaultInstance();
}
public org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock build() {
org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock result = buildPartial();
if (!result.isInitialized()) {
throw newUninitializedMessageException(result);
}
return result;
}
public org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock buildPartial() {
org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock result = new org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock(this);
int from_bitField0_ = bitField0_;
int to_bitField0_ = 0;
if (((from_bitField0_ & 0x00000001) == 0x00000001)) {
to_bitField0_ |= 0x00000001;
}
result.lockOwner_ = lockOwner_;
result.bitField0_ = to_bitField0_;
onBuilt();
return result;
}
public Builder mergeFrom(com.google.protobuf.Message other) {
if (other instanceof org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock) {
return mergeFrom((org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock)other);
} else {
super.mergeFrom(other);
return this;
}
}
public Builder mergeFrom(org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock other) {
if (other == org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock.getDefaultInstance()) return this;
if (other.hasLockOwner()) {
bitField0_ |= 0x00000001;
lockOwner_ = other.lockOwner_;
onChanged();
}
this.mergeUnknownFields(other.getUnknownFields());
return this;
}
public final boolean isInitialized() {
if (!hasLockOwner()) {
return false;
}
return true;
}
public Builder mergeFrom(
com.google.protobuf.CodedInputStream input,
com.google.protobuf.ExtensionRegistryLite extensionRegistry)
throws java.io.IOException {
org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock parsedMessage = null;
try {
parsedMessage = PARSER.parsePartialFrom(input, extensionRegistry);
} catch (com.google.protobuf.InvalidProtocolBufferException e) {
parsedMessage = (org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos.ReplicationLock) e.getUnfinishedMessage();
throw e;
} finally {
if (parsedMessage != null) {
mergeFrom(parsedMessage);
}
}
return this;
}
private int bitField0_;
// required string lock_owner = 1;
private java.lang.Object lockOwner_ = "";
/**
* <code>required string lock_owner = 1;</code>
*/
public boolean hasLockOwner() {
return ((bitField0_ & 0x00000001) == 0x00000001);
}
/**
* <code>required string lock_owner = 1;</code>
*/
public java.lang.String getLockOwner() {
java.lang.Object ref = lockOwner_;
if (!(ref instanceof java.lang.String)) {
java.lang.String s = ((com.google.protobuf.ByteString) ref)
.toStringUtf8();
lockOwner_ = s;
return s;
} else {
return (java.lang.String) ref;
}
}
/**
* <code>required string lock_owner = 1;</code>
*/
public com.google.protobuf.ByteString
getLockOwnerBytes() {
java.lang.Object ref = lockOwner_;
if (ref instanceof String) {
com.google.protobuf.ByteString b =
com.google.protobuf.ByteString.copyFromUtf8(
(java.lang.String) ref);
lockOwner_ = b;
return b;
} else {
return (com.google.protobuf.ByteString) ref;
}
}
/**
* <code>required string lock_owner = 1;</code>
*/
public Builder setLockOwner(
java.lang.String value) {
if (value == null) {
throw new NullPointerException();
}
bitField0_ |= 0x00000001;
lockOwner_ = value;
onChanged();
return this;
}
/**
* <code>required string lock_owner = 1;</code>
*/
public Builder clearLockOwner() {
bitField0_ = (bitField0_ & ~0x00000001);
lockOwner_ = getDefaultInstance().getLockOwner();
onChanged();
return this;
}
/**
* <code>required string lock_owner = 1;</code>
*/
public Builder setLockOwnerBytes(
com.google.protobuf.ByteString value) {
if (value == null) {
throw new NullPointerException();
}
bitField0_ |= 0x00000001;
lockOwner_ = value;
onChanged();
return this;
}
// @@protoc_insertion_point(builder_scope:hbase.pb.ReplicationLock)
}
static {
defaultInstance = new ReplicationLock(true);
defaultInstance.initFields();
}
// @@protoc_insertion_point(class_scope:hbase.pb.ReplicationLock)
}
public interface TableLockOrBuilder
extends com.google.protobuf.MessageOrBuilder {
@ -9934,11 +9407,6 @@ public final class ZooKeeperProtos {
private static
com.google.protobuf.GeneratedMessage.FieldAccessorTable
internal_static_hbase_pb_ReplicationHLogPosition_fieldAccessorTable;
private static com.google.protobuf.Descriptors.Descriptor
internal_static_hbase_pb_ReplicationLock_descriptor;
private static
com.google.protobuf.GeneratedMessage.FieldAccessorTable
internal_static_hbase_pb_ReplicationLock_fieldAccessorTable;
private static com.google.protobuf.Descriptors.Descriptor
internal_static_hbase_pb_TableLock_descriptor;
private static
@ -9987,15 +9455,14 @@ public final class ZooKeeperProtos {
"\006 \003(\014\"g\n\020ReplicationState\022/\n\005state\030\001 \002(\016" +
"2 .hbase.pb.ReplicationState.State\"\"\n\005St" +
"ate\022\013\n\007ENABLED\020\000\022\014\n\010DISABLED\020\001\"+\n\027Replic" +
"ationHLogPosition\022\020\n\010position\030\001 \002(\003\"%\n\017R",
"eplicationLock\022\022\n\nlock_owner\030\001 \002(\t\"\252\001\n\tT" +
"ableLock\022\'\n\ntable_name\030\001 \001(\0132\023.hbase.pb." +
"TableName\022(\n\nlock_owner\030\002 \001(\0132\024.hbase.pb" +
".ServerName\022\021\n\tthread_id\030\003 \001(\003\022\021\n\tis_sha" +
"red\030\004 \001(\010\022\017\n\007purpose\030\005 \001(\t\022\023\n\013create_tim" +
"e\030\006 \001(\003\"\036\n\013SwitchState\022\017\n\007enabled\030\001 \001(\010B" +
"E\n*org.apache.hadoop.hbase.protobuf.gene" +
"ratedB\017ZooKeeperProtosH\001\210\001\001\240\001\001"
"ationHLogPosition\022\020\n\010position\030\001 \002(\003\"\252\001\n\t",
"TableLock\022\'\n\ntable_name\030\001 \001(\0132\023.hbase.pb" +
".TableName\022(\n\nlock_owner\030\002 \001(\0132\024.hbase.p" +
"b.ServerName\022\021\n\tthread_id\030\003 \001(\003\022\021\n\tis_sh" +
"ared\030\004 \001(\010\022\017\n\007purpose\030\005 \001(\t\022\023\n\013create_ti" +
"me\030\006 \001(\003\"\036\n\013SwitchState\022\017\n\007enabled\030\001 \001(\010" +
"BE\n*org.apache.hadoop.hbase.protobuf.gen" +
"eratedB\017ZooKeeperProtosH\001\210\001\001\240\001\001"
};
com.google.protobuf.Descriptors.FileDescriptor.InternalDescriptorAssigner assigner =
new com.google.protobuf.Descriptors.FileDescriptor.InternalDescriptorAssigner() {
@ -10056,20 +9523,14 @@ public final class ZooKeeperProtos {
com.google.protobuf.GeneratedMessage.FieldAccessorTable(
internal_static_hbase_pb_ReplicationHLogPosition_descriptor,
new java.lang.String[] { "Position", });
internal_static_hbase_pb_ReplicationLock_descriptor =
getDescriptor().getMessageTypes().get(9);
internal_static_hbase_pb_ReplicationLock_fieldAccessorTable = new
com.google.protobuf.GeneratedMessage.FieldAccessorTable(
internal_static_hbase_pb_ReplicationLock_descriptor,
new java.lang.String[] { "LockOwner", });
internal_static_hbase_pb_TableLock_descriptor =
getDescriptor().getMessageTypes().get(10);
getDescriptor().getMessageTypes().get(9);
internal_static_hbase_pb_TableLock_fieldAccessorTable = new
com.google.protobuf.GeneratedMessage.FieldAccessorTable(
internal_static_hbase_pb_TableLock_descriptor,
new java.lang.String[] { "TableName", "LockOwner", "ThreadId", "IsShared", "Purpose", "CreateTime", });
internal_static_hbase_pb_SwitchState_descriptor =
getDescriptor().getMessageTypes().get(11);
getDescriptor().getMessageTypes().get(10);
internal_static_hbase_pb_SwitchState_fieldAccessorTable = new
com.google.protobuf.GeneratedMessage.FieldAccessorTable(
internal_static_hbase_pb_SwitchState_descriptor,

View File

@ -142,13 +142,6 @@ message ReplicationHLogPosition {
required int64 position = 1;
}
/**
* Used by replication. Used to lock a region server during failover.
*/
message ReplicationLock {
required string lock_owner = 1;
}
/**
* Metadata associated with a table lock in zookeeper
*/

View File

@ -75,9 +75,6 @@ public class TestRSGroups extends TestRSGroupsBase {
RSGroupBasedLoadBalancer.class.getName());
TEST_UTIL.getConfiguration().set(CoprocessorHost.MASTER_COPROCESSOR_CONF_KEY,
RSGroupAdminEndpoint.class.getName());
TEST_UTIL.getConfiguration().setBoolean(
HConstants.ZOOKEEPER_USEMULTI,
true);
TEST_UTIL.startMiniCluster(NUM_SLAVES_BASE);
TEST_UTIL.getConfiguration().set(
ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART,

View File

@ -100,7 +100,6 @@ import org.apache.hadoop.hbase.master.balancer.LoadBalancerFactory;
import org.apache.hadoop.hbase.master.cleaner.HFileCleaner;
import org.apache.hadoop.hbase.master.cleaner.LogCleaner;
import org.apache.hadoop.hbase.master.cleaner.ReplicationMetaCleaner;
import org.apache.hadoop.hbase.master.cleaner.ReplicationZKLockCleanerChore;
import org.apache.hadoop.hbase.master.normalizer.NormalizationPlan;
import org.apache.hadoop.hbase.master.normalizer.NormalizationPlan.PlanType;
import org.apache.hadoop.hbase.master.normalizer.RegionNormalizer;
@ -314,7 +313,6 @@ public class HMaster extends HRegionServer implements MasterServices {
private PeriodicDoMetrics periodicDoMetricsChore = null;
CatalogJanitor catalogJanitorChore;
private ReplicationZKLockCleanerChore replicationZKLockCleanerChore;
private ReplicationMetaCleaner replicationMetaCleaner;
private LogCleaner logCleaner;
private HFileCleaner hfileCleaner;
@ -984,17 +982,7 @@ public class HMaster extends HRegionServer implements MasterServices {
if (LOG.isTraceEnabled()) {
LOG.trace("Started service threads");
}
if (conf.getClass("hbase.region.replica.replication.replicationQueues.class",
ReplicationFactory.defaultReplicationQueueClass) == ReplicationQueuesZKImpl.class && !conf
.getBoolean(HConstants.ZOOKEEPER_USEMULTI, true)) {
try {
replicationZKLockCleanerChore = new ReplicationZKLockCleanerChore(this, this,
cleanerInterval, this.getZooKeeper(), this.conf);
getChoreService().scheduleChore(replicationZKLockCleanerChore);
} catch (Exception e) {
LOG.error("start replicationZKLockCleanerChore failed", e);
}
}
replicationMetaCleaner = new ReplicationMetaCleaner(this, this, cleanerInterval);
getChoreService().scheduleChore(replicationMetaCleaner);
}
@ -1030,7 +1018,6 @@ public class HMaster extends HRegionServer implements MasterServices {
// Clean up and close up shop
if (this.logCleaner != null) this.logCleaner.cancel(true);
if (this.hfileCleaner != null) this.hfileCleaner.cancel(true);
if (this.replicationZKLockCleanerChore != null) this.replicationZKLockCleanerChore.cancel(true);
if (this.replicationMetaCleaner != null) this.replicationMetaCleaner.cancel(true);
if (this.quotaManager != null) this.quotaManager.stop();
if (this.activeMasterManager != null) this.activeMasterManager.stop();

View File

@ -1,112 +0,0 @@
/**
*
* 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.cleaner;
import com.google.common.annotations.VisibleForTesting;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hadoop.hbase.ScheduledChore;
import org.apache.hadoop.hbase.Stoppable;
import org.apache.hadoop.hbase.classification.InterfaceAudience;
import org.apache.hadoop.hbase.replication.ReplicationFactory;
import org.apache.hadoop.hbase.replication.ReplicationQueuesZKImpl;
import org.apache.hadoop.hbase.replication.ReplicationTracker;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.data.Stat;
/**
* A cleaner that cleans replication locks on zk which is locked by dead region servers
*/
@InterfaceAudience.Private
public class ReplicationZKLockCleanerChore extends ScheduledChore {
private static final Log LOG = LogFactory.getLog(ReplicationZKLockCleanerChore.class);
private ZooKeeperWatcher zk;
private ReplicationTracker tracker;
private long ttl;
private ReplicationQueuesZKImpl queues;
// Wait some times before delete lock to prevent a session expired RS not dead fully.
private static final long DEFAULT_TTL = 60 * 10 * 1000;//10 min
@VisibleForTesting
public static final String TTL_CONFIG_KEY = "hbase.replication.zk.deadrs.lock.ttl";
public ReplicationZKLockCleanerChore(Stoppable stopper, Abortable abortable, int period,
ZooKeeperWatcher zk, Configuration conf) throws Exception {
super("ReplicationZKLockCleanerChore", stopper, period);
this.zk = zk;
this.ttl = conf.getLong(TTL_CONFIG_KEY, DEFAULT_TTL);
tracker = ReplicationFactory.getReplicationTracker(zk,
ReplicationFactory.getReplicationPeers(zk, conf, abortable), conf, abortable, stopper);
queues = new ReplicationQueuesZKImpl(zk, conf, abortable);
}
@Override protected void chore() {
try {
List<String> regionServers = tracker.getListOfRegionServers();
if (regionServers == null) {
return;
}
Set<String> rsSet = new HashSet<String>(regionServers);
List<String> replicators = queues.getListOfReplicators();
for (String replicator: replicators) {
try {
String lockNode = queues.getLockZNode(replicator);
byte[] data = ZKUtil.getData(zk, lockNode);
if (data == null) {
continue;
}
String rsServerNameZnode = Bytes.toString(data);
String[] array = rsServerNameZnode.split("/");
String znode = array[array.length - 1];
if (!rsSet.contains(znode)) {
Stat s = zk.getRecoverableZooKeeper().exists(lockNode, false);
if (s != null && EnvironmentEdgeManager.currentTime() - s.getMtime() > this.ttl) {
// server is dead, but lock is still there, we have to delete the lock.
ZKUtil.deleteNode(zk, lockNode);
LOG.info("Remove lock acquired by dead RS: " + lockNode + " by " + znode);
}
continue;
}
LOG.info("Skip lock acquired by live RS: " + lockNode + " by " + znode);
} catch (KeeperException.NoNodeException ignore) {
} catch (InterruptedException e) {
LOG.warn("zk operation interrupted", e);
Thread.currentThread().interrupt();
}
}
} catch (KeeperException e) {
LOG.warn("zk operation interrupted", e);
}
}
}

View File

@ -45,7 +45,6 @@ import org.apache.hadoop.hbase.client.Result;
import org.apache.hadoop.hbase.client.Table;
import org.apache.hadoop.hbase.client.replication.ReplicationAdmin;
import org.apache.hadoop.hbase.coprocessor.CoprocessorHost;
import org.apache.hadoop.hbase.master.cleaner.ReplicationZKLockCleanerChore;
import org.apache.hadoop.hbase.regionserver.HRegion;
import org.apache.hadoop.hbase.regionserver.HRegionServer;
import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
@ -97,12 +96,9 @@ public class TestMultiSlaveReplication {
conf1.setLong(HConstants.THREAD_WAKE_FREQUENCY, 100);
conf1.setStrings(CoprocessorHost.USER_REGION_COPROCESSOR_CONF_KEY,
"org.apache.hadoop.hbase.replication.TestMasterReplication$CoprocessorCounter");
conf1.setBoolean(HConstants.ZOOKEEPER_USEMULTI , false);// for testZKLockCleaner
conf1.setInt("hbase.master.cleaner.interval", 5 * 1000);
conf1.setClass("hbase.region.replica.replication.replicationQueues.class",
ReplicationQueuesZKImpl.class, ReplicationQueues.class);
conf1.setLong(ReplicationZKLockCleanerChore.TTL_CONFIG_KEY, 0L);
utility1 = new HBaseTestingUtility(conf1);
utility1.startMiniZKCluster();
@ -210,40 +206,6 @@ public class TestMultiSlaveReplication {
utility1.shutdownMiniCluster();
}
@Test
public void testZKLockCleaner() throws Exception {
MiniHBaseCluster cluster = utility1.startMiniCluster(1, 2);
HBaseAdmin admin = utility1.getHBaseAdmin();
HTableDescriptor table = new HTableDescriptor(TableName.valueOf(Bytes.toBytes("zk")));
HColumnDescriptor fam = new HColumnDescriptor(famName);
fam.setScope(HConstants.REPLICATION_SCOPE_GLOBAL);
table.addFamily(fam);
admin.createTable(table);
ReplicationAdmin replicationAdmin = new ReplicationAdmin(conf1);
ReplicationPeerConfig rpc = new ReplicationPeerConfig();
rpc.setClusterKey(utility2.getClusterKey());
replicationAdmin.addPeer("cluster2", rpc, null);
HRegionServer rs = cluster.getRegionServer(0);
ReplicationQueuesZKImpl zk = new ReplicationQueuesZKImpl(rs.getZooKeeper(), conf1, rs);
zk.init(rs.getServerName().toString());
List<String> replicators = zk.getListOfReplicators();
assertEquals(2, replicators.size());
String zNode = cluster.getRegionServer(1).getServerName().toString();
assertTrue(zk.lockOtherRS(zNode));
assertTrue(zk.checkLockExists(zNode));
Thread.sleep(10000);
assertTrue(zk.checkLockExists(zNode));
cluster.abortRegionServer(0);
Thread.sleep(10000);
HRegionServer rs1 = cluster.getRegionServer(1);
zk = new ReplicationQueuesZKImpl(rs1.getZooKeeper(), conf1, rs1);
zk.init(rs1.getServerName().toString());
assertFalse(zk.checkLockExists(zNode));
utility1.shutdownMiniCluster();
}
private void rollWALAndWait(final HBaseTestingUtility utility, final TableName table,
final byte[] row) throws IOException {
final Admin admin = utility.getHBaseAdmin();

View File

@ -287,7 +287,6 @@ public abstract class TestReplicationSourceManager {
@Test
public void testClaimQueues() throws Exception {
conf.setBoolean(HConstants.ZOOKEEPER_USEMULTI, true);
final Server server = new DummyServer("hostname0.example.org");

View File

@ -62,7 +62,6 @@ public class TestReplicationSourceManagerZkImpl extends TestReplicationSourceMan
// Tests the naming convention of adopted queues for ReplicationQueuesZkImpl
@Test
public void testNodeFailoverDeadServerParsing() throws Exception {
conf.setBoolean(HConstants.ZOOKEEPER_USEMULTI, true);
final Server server = new DummyServer("ec2-54-234-230-108.compute-1.amazonaws.com");
ReplicationQueues repQueues =
ReplicationFactory.getReplicationQueues(new ReplicationQueuesArguments(conf, server,
@ -117,7 +116,6 @@ public class TestReplicationSourceManagerZkImpl extends TestReplicationSourceMan
@Test
public void testFailoverDeadServerCversionChange() throws Exception {
conf.setBoolean(HConstants.ZOOKEEPER_USEMULTI, true);
final Server s0 = new DummyServer("cversion-change0.example.org");
ReplicationQueues repQueues =
ReplicationFactory.getReplicationQueues(new ReplicationQueuesArguments(conf, s0,

View File

@ -59,7 +59,6 @@ public class TestZKMulti {
public static void setUpBeforeClass() throws Exception {
TEST_UTIL.startMiniZKCluster();
Configuration conf = TEST_UTIL.getConfiguration();
conf.setBoolean("hbase.zookeeper.useMulti", true);
Abortable abortable = new Abortable() {
@Override
public void abort(String why, Throwable e) {
@ -313,32 +312,6 @@ public class TestZKMulti {
assertTrue("Failed to delete child znodes!", 0 == children.size());
}
/**
* Verifies that for the given root node, it should delete all the child nodes
* recursively using normal sequential way.
*/
@Test (timeout=60000)
public void testdeleteChildrenRecursivelySequential() throws Exception {
String parentZNode = "/testRootSeq";
createZNodeTree(parentZNode);
boolean useMulti = zkw.getConfiguration().getBoolean(
"hbase.zookeeper.useMulti", false);
zkw.getConfiguration().setBoolean("hbase.zookeeper.useMulti", false);
try {
// disables the multi-update api execution
ZKUtil.deleteChildrenRecursivelyMultiOrSequential(zkw, true, parentZNode);
assertTrue("Wrongly deleted parent znode!",
ZKUtil.checkExists(zkw, parentZNode) > -1);
List<String> children = zkw.getRecoverableZooKeeper().getChildren(
parentZNode, false);
assertTrue("Failed to delete child znodes!", 0 == children.size());
} finally {
// sets back the multi-update api execution
zkw.getConfiguration().setBoolean("hbase.zookeeper.useMulti", useMulti);
}
}
/**
* Verifies that for the given root node, it should delete all the nodes recursively using
* multi-update api.
@ -352,26 +325,6 @@ public class TestZKMulti {
assertTrue("Parent znode should be deleted.", ZKUtil.checkExists(zkw, parentZNode) == -1);
}
/**
* Verifies that for the given root node, it should delete all the nodes recursively using
* normal sequential way.
*/
@Test(timeout = 60000)
public void testDeleteNodeRecursivelySequential() throws Exception {
String parentZNode = "/testdeleteNodeRecursivelySequential";
createZNodeTree(parentZNode);
boolean useMulti = zkw.getConfiguration().getBoolean("hbase.zookeeper.useMulti", false);
zkw.getConfiguration().setBoolean("hbase.zookeeper.useMulti", false);
try {
// disables the multi-update api execution
ZKUtil.deleteNodeRecursively(zkw, parentZNode);
assertTrue("Parent znode should be deleted.", ZKUtil.checkExists(zkw, parentZNode) == -1);
} finally {
// sets back the multi-update api execution
zkw.getConfiguration().setBoolean("hbase.zookeeper.useMulti", useMulti);
}
}
@Test(timeout = 60000)
public void testDeleteNodeRecursivelyMultiOrSequential() throws Exception {
String parentZNode1 = "/testdeleteNode1";

View File

@ -70,9 +70,6 @@ public class TestShellRSGroups {
TEST_UTIL.getConfiguration().set(
CoprocessorHost.MASTER_COPROCESSOR_CONF_KEY,
RSGroupAdminEndpoint.class.getName());
TEST_UTIL.getConfiguration().setBoolean(
HConstants.ZOOKEEPER_USEMULTI,
true);
TEST_UTIL.startMiniCluster(1,4);

View File

@ -384,8 +384,9 @@ See also <<casestudies.max.transfer.threads,casestudies.max.transfer.threads>> a
=== ZooKeeper Requirements
ZooKeeper 3.4.x is required as of HBase 1.0.0.
HBase makes use of the `multi` functionality that is only available since 3.4.0 (The `useMulti` configuration option defaults to `true` in HBase 1.0.0).
See link:https://issues.apache.org/jira/browse/HBASE-12241[HBASE-12241 (The crash of regionServer when taking deadserver's replication queue breaks replication)] and link:https://issues.apache.org/jira/browse/HBASE-6775[HBASE-6775 (Use ZK.multi when available for HBASE-6710 0.92/0.94 compatibility fix)] for background.
HBase makes use of the `multi` functionality that is only available since Zookeeper 3.4.0. The `hbase.zookeeper.useMulti` configuration property defaults to `true` in HBase 1.0.0.
Refer to link:https://issues.apache.org/jira/browse/HBASE-12241[HBASE-12241 (The crash of regionServer when taking deadserver's replication queue breaks replication)] and link:https://issues.apache.org/jira/browse/HBASE-6775[HBASE-6775 (Use ZK.multi when available for HBASE-6710 0.92/0.94 compatibility fix)] for background.
The property is deprecated and useMulti is always enabled in HBase 2.0.
[[standalone_dist]]
== HBase run modes: Standalone and Distributed

View File

@ -100,9 +100,7 @@ In the example below we have ZooKeeper persist to _/user/local/zookeeper_.
.What version of ZooKeeper should I use?
[CAUTION]
====
The newer version, the better.
For example, some folks have been bitten by link:https://issues.apache.org/jira/browse/ZOOKEEPER-1277[ZOOKEEPER-1277].
If running zookeeper 3.5+, you can ask hbase to make use of the new multi operation by enabling <<hbase.zookeeper.useMulti,hbase.zookeeper.useMulti>>" in your _hbase-site.xml_.
The newer version, the better. ZooKeeper 3.4.x is required as of HBase 1.0.0
====
.ZooKeeper Maintenance