HBASE-19603 Fixed Checkstyle errors in hbase-replication and enabled Checkstyle to fail on violations

This commit is contained in:
Jan Hentschel 2017-12-23 12:37:51 +01:00
parent 022a348d61
commit 228d7a5a46
14 changed files with 116 additions and 95 deletions

View File

@ -69,6 +69,22 @@
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-source-plugin</artifactId>
</plugin>
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-checkstyle-plugin</artifactId>
<executions>
<execution>
<id>checkstyle</id>
<phase>validate</phase>
<goals>
<goal>check</goal>
</goals>
<configuration>
<failOnViolation>true</failOnViolation>
</configuration>
</execution>
</executions>
</plugin>
</plugins>
<pluginManagement>
<plugins>

View File

@ -19,17 +19,19 @@
package org.apache.hadoop.hbase.replication;
import org.apache.commons.lang3.reflect.ConstructorUtils;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hadoop.hbase.Stoppable;
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
import org.apache.yetus.audience.InterfaceAudience;
/**
* A factory class for instantiating replication objects that deal with replication state.
*/
@InterfaceAudience.Private
public class ReplicationFactory {
public final class ReplicationFactory {
private ReplicationFactory() {
}
public static ReplicationQueues getReplicationQueues(ReplicationQueuesArguments args)
throws Exception {
@ -49,7 +51,7 @@ public class ReplicationFactory {
}
public static ReplicationPeers getReplicationPeers(final ZKWatcher zk, Configuration conf,
final ReplicationQueuesClient queuesClient, Abortable abortable) {
final ReplicationQueuesClient queuesClient, Abortable abortable) {
return new ReplicationPeersZKImpl(zk, conf, queuesClient, abortable);
}

View File

@ -31,17 +31,17 @@ import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.client.replication.ReplicationPeerConfigUtil;
import org.apache.hadoop.hbase.exceptions.DeserializationException;
import org.apache.hadoop.hbase.log.HBaseMarkers;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
import org.apache.hadoop.hbase.zookeeper.ZKNodeTracker;
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.KeeperException.NodeExistsException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
@InterfaceAudience.Private
public class ReplicationPeerZKImpl extends ReplicationStateZKBase
implements ReplicationPeer, Abortable, Closeable {
@ -77,10 +77,9 @@ public class ReplicationPeerZKImpl extends ReplicationStateZKBase
* start a state tracker to check whether this peer is enabled or not
*
* @param peerStateNode path to zk node which stores peer state
* @throws KeeperException
* @throws KeeperException if creating the znode fails
*/
public void startStateTracker(String peerStateNode)
throws KeeperException {
public void startStateTracker(String peerStateNode) throws KeeperException {
ensurePeerEnabled(peerStateNode);
this.peerStateTracker = new PeerStateTracker(peerStateNode, zookeeper, this);
this.peerStateTracker.start();
@ -101,10 +100,8 @@ public class ReplicationPeerZKImpl extends ReplicationStateZKBase
/**
* start a table-cfs tracker to listen the (table, cf-list) map change
* @param peerConfigNode path to zk node which stores table-cfs
* @throws KeeperException
*/
public void startPeerConfigTracker(String peerConfigNode)
throws KeeperException {
public void startPeerConfigTracker(String peerConfigNode) throws KeeperException {
this.peerConfigTracker = new PeerConfigTracker(peerConfigNode, zookeeper,
this);
this.peerConfigTracker.start();
@ -208,7 +205,7 @@ public class ReplicationPeerZKImpl extends ReplicationStateZKBase
* Parse the raw data from ZK to get a peer's state
* @param bytes raw ZK data
* @return True if the passed in <code>bytes</code> are those of a pb serialized ENABLED state.
* @throws DeserializationException
* @throws DeserializationException if parsing the state fails
*/
public static boolean isStateEnabled(final byte[] bytes) throws DeserializationException {
ReplicationProtos.ReplicationState.State state = parseStateFrom(bytes);
@ -218,7 +215,7 @@ public class ReplicationPeerZKImpl extends ReplicationStateZKBase
/**
* @param bytes Content of a state znode.
* @return State parsed from the passed bytes.
* @throws DeserializationException
* @throws DeserializationException if a ProtoBuf operation fails
*/
private static ReplicationProtos.ReplicationState.State parseStateFrom(final byte[] bytes)
throws DeserializationException {
@ -240,11 +237,9 @@ public class ReplicationPeerZKImpl extends ReplicationStateZKBase
* Utility method to ensure an ENABLED znode is in place; if not present, we create it.
* @param path Path to znode to check
* @return True if we created the znode.
* @throws NodeExistsException
* @throws KeeperException
* @throws KeeperException if creating the znode fails
*/
private boolean ensurePeerEnabled(final String path)
throws NodeExistsException, KeeperException {
private boolean ensurePeerEnabled(final String path) throws KeeperException {
if (ZKUtil.checkExists(zookeeper, path) == -1) {
// There is a race b/w PeerWatcher and ReplicationZookeeper#add method to create the
// peer-state znode. This happens while adding a peer.

View File

@ -25,8 +25,8 @@ import java.util.Set;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.TableName;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.hadoop.hbase.util.Pair;
import org.apache.yetus.audience.InterfaceAudience;
/**
* This provides an interface for maintaining a set of peer clusters. These peers are remote slave
@ -76,7 +76,7 @@ public interface ReplicationPeers {
* newly connected cluster.
* @param peerId a short that identifies the cluster
* @return whether a ReplicationPeer was successfully created
* @throws ReplicationException
* @throws ReplicationException if connecting to the peer fails
*/
boolean peerConnected(String peerId) throws ReplicationException;
@ -182,7 +182,7 @@ public interface ReplicationPeers {
* Update the peerConfig for the a given peer cluster
* @param id a short that identifies the cluster
* @param peerConfig new config for the peer cluster
* @throws ReplicationException
* @throws ReplicationException if updating the peer configuration fails
*/
void updatePeerConfig(String id, ReplicationPeerConfig peerConfig) throws ReplicationException;
}

View File

@ -35,7 +35,6 @@ import org.apache.hadoop.hbase.HBaseConfiguration;
import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.client.replication.ReplicationPeerConfigUtil;
import org.apache.hadoop.hbase.exceptions.DeserializationException;
import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
import org.apache.hadoop.hbase.replication.ReplicationPeer.PeerState;
import org.apache.hadoop.hbase.util.Pair;
import org.apache.hadoop.hbase.zookeeper.ZKConfig;
@ -48,6 +47,8 @@ import org.apache.zookeeper.KeeperException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
/**
* This class provides an implementation of the ReplicationPeers interface using ZooKeeper. The
* peers znode contains a list of all peer replication clusters and the current replication state of
@ -462,8 +463,8 @@ public class ReplicationPeersZKImpl extends ReplicationStateZKBase implements Re
/**
* Update the state znode of a peer cluster.
* @param id
* @param state
* @param id the id of the peer
* @param state the state to update to
*/
private void changePeerState(String id, ReplicationProtos.ReplicationState.State state)
throws ReplicationException {
@ -491,7 +492,7 @@ public class ReplicationPeersZKImpl extends ReplicationStateZKBase implements Re
* Helper method to connect to a peer
* @param peerId peer's identifier
* @return object representing the peer
* @throws ReplicationException
* @throws ReplicationException if creating the peer fails
*/
private ReplicationPeerZKImpl createPeer(String peerId) throws ReplicationException {
Pair<ReplicationPeerConfig, Configuration> pair = getPeerConf(peerId);
@ -520,7 +521,10 @@ public class ReplicationPeersZKImpl extends ReplicationStateZKBase implements Re
}
private void checkQueuesDeleted(String peerId) throws ReplicationException {
if (queuesClient == null) return;
if (queuesClient == null) {
return;
}
try {
List<String> replicators = queuesClient.getListOfReplicators();
if (replicators == null || replicators.isEmpty()) {

View File

@ -18,15 +18,14 @@
*/
package org.apache.hadoop.hbase.replication;
import java.util.ArrayList;
import java.util.Collections;
import java.util.List;
import org.apache.hadoop.hbase.ServerName;
import org.apache.yetus.audience.InterfaceAudience;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hadoop.hbase.ServerName;
/**
* This class is responsible for the parsing logic for a znode representing a queue.
@ -66,8 +65,9 @@ public class ReplicationQueueInfo {
*/
private static void
extractDeadServersFromZNodeString(String deadServerListStr, List<ServerName> result) {
if(deadServerListStr == null || result == null || deadServerListStr.isEmpty()) return;
if(deadServerListStr == null || result == null || deadServerListStr.isEmpty()) {
return;
}
// valid server name delimiter "-" has to be after "," in a server name
int seenCommaCnt = 0;
@ -76,25 +76,25 @@ public class ReplicationQueueInfo {
for (int i = 0; i < len; i++) {
switch (deadServerListStr.charAt(i)) {
case ',':
seenCommaCnt += 1;
break;
case '-':
if(seenCommaCnt>=2) {
if (i > startIndex) {
String serverName = deadServerListStr.substring(startIndex, i);
if(ServerName.isFullServerName(serverName)){
result.add(ServerName.valueOf(serverName));
} else {
LOG.error("Found invalid server name:" + serverName);
case ',':
seenCommaCnt += 1;
break;
case '-':
if(seenCommaCnt>=2) {
if (i > startIndex) {
String serverName = deadServerListStr.substring(startIndex, i);
if(ServerName.isFullServerName(serverName)){
result.add(ServerName.valueOf(serverName));
} else {
LOG.error("Found invalid server name:" + serverName);
}
startIndex = i + 1;
}
startIndex = i + 1;
seenCommaCnt = 0;
}
seenCommaCnt = 0;
}
break;
default:
break;
break;
default:
break;
}
}

View File

@ -22,8 +22,8 @@ import java.util.List;
import java.util.SortedSet;
import org.apache.hadoop.fs.Path;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.hadoop.hbase.util.Pair;
import org.apache.yetus.audience.InterfaceAudience;
/**
* This provides an interface for maintaining a region server's replication queues. These queues
@ -90,7 +90,8 @@ public interface ReplicationQueues {
/**
* Get a list of all queues for this region server.
* @return a list of queueIds, an empty list if this region server is dead and has no outstanding queues
* @return a list of queueIds, an empty list if this region server is dead and has no outstanding
* queues
*/
List<String> getAllQueues();
@ -111,7 +112,7 @@ public interface ReplicationQueues {
/**
* Remove the znode of region server if the queue is empty.
* @param regionserver
* @param regionserver the id of the region server
*/
void removeReplicatorIfQueueIsEmpty(String regionserver);

View File

@ -67,7 +67,7 @@ public interface ReplicationQueuesClient {
* is concurrent queue failover. However, some newly created WALs during the call may
* not be included.
*/
Set<String> getAllWALs() throws KeeperException;
Set<String> getAllWALs() throws KeeperException;
/**
* Get the change version number of replication hfile references node. This can be used as

View File

@ -23,8 +23,6 @@ import java.util.Set;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableSet;
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
import org.apache.hadoop.hbase.zookeeper.ZNodePaths;
@ -34,6 +32,9 @@ import org.apache.zookeeper.data.Stat;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableSet;
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
@InterfaceAudience.Private
public class ReplicationQueuesClientZKImpl extends ReplicationStateZKBase implements
ReplicationQueuesClient {

View File

@ -302,8 +302,9 @@ public class ReplicationQueuesZKImpl extends ReplicationStateZKBase implements R
// add delete op for peer
listOfOps.add(ZKUtilOp.deleteNodeFailSilent(oldClusterZnode));
if (LOG.isTraceEnabled())
if (LOG.isTraceEnabled()) {
LOG.trace(" The multi list size is: " + listOfOps.size());
}
}
ZKUtil.multiOrSequential(this.zookeeper, listOfOps, false);

View File

@ -24,10 +24,6 @@ import java.util.List;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.hbase.thirdparty.com.google.protobuf.CodedOutputStream;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
import org.apache.hadoop.hbase.zookeeper.ZKConfig;
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
@ -35,6 +31,12 @@ import org.apache.hadoop.hbase.zookeeper.ZNodePaths;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.zookeeper.KeeperException;
import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
import org.apache.hbase.thirdparty.com.google.protobuf.CodedOutputStream;
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
import org.apache.hadoop.hbase.shaded.protobuf.generated.ReplicationProtos;
/**
* This is a base class for maintaining replication state in zookeeper.
*/
@ -104,7 +106,7 @@ public abstract class ReplicationStateZKBase {
}
/**
* @param state
* @param state the state to convert into a byte array
* @return Serialized protobuf of <code>state</code> with pb magic prefix prepended suitable for
* use as content of a peer-state znode under a peer cluster id as in
* /hbase/replication/peers/PEER_ID/peer-state.

View File

@ -18,32 +18,6 @@
*/
package org.apache.hadoop.hbase.replication;
import org.apache.hadoop.hbase.CompareOperator;
import org.apache.hbase.thirdparty.com.google.common.util.concurrent.ThreadFactoryBuilder;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hadoop.hbase.HColumnDescriptor;
import org.apache.hadoop.hbase.HConstants;
import org.apache.hadoop.hbase.HTableDescriptor;
import org.apache.hadoop.hbase.NamespaceDescriptor;
import org.apache.hadoop.hbase.TableExistsException;
import org.apache.hadoop.hbase.TableName;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.hadoop.hbase.client.Admin;
import org.apache.hadoop.hbase.client.Connection;
import org.apache.hadoop.hbase.client.ConnectionFactory;
import org.apache.hadoop.hbase.client.Get;
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.client.Table;
import org.apache.hadoop.hbase.filter.CompareFilter;
import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
import org.apache.hadoop.hbase.regionserver.BloomType;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.RetryCounter;
import org.apache.hadoop.hbase.util.RetryCounterFactory;
import java.io.IOException;
import java.io.InterruptedIOException;
import java.util.ArrayList;
@ -58,6 +32,32 @@ import java.util.concurrent.LinkedBlockingQueue;
import java.util.concurrent.ThreadPoolExecutor;
import java.util.concurrent.TimeUnit;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hadoop.hbase.CompareOperator;
import org.apache.hadoop.hbase.HColumnDescriptor;
import org.apache.hadoop.hbase.HConstants;
import org.apache.hadoop.hbase.HTableDescriptor;
import org.apache.hadoop.hbase.NamespaceDescriptor;
import org.apache.hadoop.hbase.TableExistsException;
import org.apache.hadoop.hbase.TableName;
import org.apache.hadoop.hbase.client.Admin;
import org.apache.hadoop.hbase.client.Connection;
import org.apache.hadoop.hbase.client.ConnectionFactory;
import org.apache.hadoop.hbase.client.Get;
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.client.Table;
import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
import org.apache.hadoop.hbase.regionserver.BloomType;
import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.RetryCounter;
import org.apache.hadoop.hbase.util.RetryCounterFactory;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.hbase.thirdparty.com.google.common.util.concurrent.ThreadFactoryBuilder;
/*
* Abstract class that provides an interface to the Replication Table. Which is currently
* being used for WAL offset tracking.
@ -321,12 +321,12 @@ abstract class ReplicationTableBase {
*
* @param server name of the server
* @return a ResultScanner over the QueueIds belonging to the server
* @throws IOException
* @throws IOException if getting the table or the scanner fails
*/
protected ResultScanner getQueuesBelongingToServer(String server) throws IOException {
Scan scan = new Scan();
SingleColumnValueFilter filterMyQueues = new SingleColumnValueFilter(CF_QUEUE, COL_QUEUE_OWNER,
CompareOperator.EQUAL, Bytes.toBytes(server));
CompareOperator.EQUAL, Bytes.toBytes(server));
scan.setFilter(filterMyQueues);
scan.addColumn(CF_QUEUE, COL_QUEUE_OWNER);
scan.addColumn(CF_QUEUE, COL_QUEUE_OWNER_HISTORY);
@ -341,7 +341,7 @@ abstract class ReplicationTableBase {
* the CreateReplicationWorker thread. It is up to the caller of this method to close the
* returned Table
* @return the Replication Table when it is created
* @throws IOException
* @throws IOException if getting the table or the scanner fails
*/
protected Table getOrBlockOnReplicationTable() throws IOException {
// Sleep until the Replication Table becomes available
@ -359,7 +359,7 @@ abstract class ReplicationTableBase {
* Creates a new copy of the Replication Table and sets up the proper Table time outs for it
*
* @return the Replication Table
* @throws IOException
* @throws IOException if getting the table fails
*/
private Table getAndSetUpReplicationTable() throws IOException {
Table replicationTable = connection.getTable(REPLICATION_TABLE_NAME);
@ -413,7 +413,7 @@ abstract class ReplicationTableBase {
* Create the replication table with the provided HColumnDescriptor REPLICATION_COL_DESCRIPTOR
* in TableBasedReplicationQueuesImpl
*
* @throws IOException
* @throws IOException if creating the table fails
*/
private void createReplicationTable() throws IOException {
HTableDescriptor replicationTableDescriptor = new HTableDescriptor(REPLICATION_TABLE_NAME);
@ -429,7 +429,6 @@ abstract class ReplicationTableBase {
* Checks whether the Replication Table exists yet
*
* @return whether the Replication Table exists
* @throws IOException
*/
private boolean replicationTableExists() {
try {

View File

@ -35,7 +35,7 @@ public interface ReplicationTracker {
/**
* Register a replication listener to receive replication events.
* @param listener
* @param listener the listener to register
*/
public void registerListener(ReplicationListener listener);

View File

@ -22,13 +22,13 @@ import java.util.ArrayList;
import java.util.List;
import java.util.concurrent.CopyOnWriteArrayList;
import org.apache.hadoop.hbase.zookeeper.ZKListener;
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.hbase.Abortable;
import org.apache.hadoop.hbase.Stoppable;
import org.apache.hadoop.hbase.zookeeper.ZKListener;
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
import org.apache.yetus.audience.InterfaceAudience;
import org.apache.zookeeper.KeeperException;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;