HBASE-15118 Fix findbugs complaint in hbase-server

This commit is contained in:
stack 2016-01-18 13:02:17 -08:00
parent 47fc696bc6
commit 9bf26f46d1
52 changed files with 325 additions and 239 deletions

View File

@ -60,7 +60,7 @@ public class JMXListener implements Coprocessor {
* only 1 JMX instance is allowed, otherwise there is port conflict even if * only 1 JMX instance is allowed, otherwise there is port conflict even if
* we only load regionserver coprocessor on master * we only load regionserver coprocessor on master
*/ */
private static JMXConnectorServer jmxCS = null; private static JMXConnectorServer JMX_CS = null;
public static JMXServiceURL buildJMXServiceURL(int rmiRegistryPort, public static JMXServiceURL buildJMXServiceURL(int rmiRegistryPort,
int rmiConnectorPort) throws IOException { int rmiConnectorPort) throws IOException {
@ -137,8 +137,13 @@ public class JMXListener implements Coprocessor {
try { try {
// Start the JMXListener with the connection string // Start the JMXListener with the connection string
jmxCS = JMXConnectorServerFactory.newJMXConnectorServer(serviceUrl, jmxEnv, mbs); synchronized(JMXListener.class) {
jmxCS.start(); if (JMX_CS != null) {
throw new RuntimeException("Started by another thread?");
}
JMX_CS = JMXConnectorServerFactory.newJMXConnectorServer(serviceUrl, jmxEnv, mbs);
JMX_CS.start();
}
LOG.info("ConnectorServer started!"); LOG.info("ConnectorServer started!");
} catch (IOException e) { } catch (IOException e) {
LOG.error("fail to start connector server!", e); LOG.error("fail to start connector server!", e);
@ -148,10 +153,10 @@ public class JMXListener implements Coprocessor {
public void stopConnectorServer() throws IOException { public void stopConnectorServer() throws IOException {
synchronized(JMXListener.class) { synchronized(JMXListener.class) {
if (jmxCS != null) { if (JMX_CS != null) {
jmxCS.stop(); JMX_CS.stop();
LOG.info("ConnectorServer stopped!"); LOG.info("ConnectorServer stopped!");
jmxCS = null; JMX_CS = null;
} }
} }
} }
@ -186,7 +191,7 @@ public class JMXListener implements Coprocessor {
} }
synchronized(JMXListener.class) { synchronized(JMXListener.class) {
if (jmxCS != null) { if (JMX_CS != null) {
LOG.info("JMXListener has been started at Registry port " + rmiRegistryPort); LOG.info("JMXListener has been started at Registry port " + rmiRegistryPort);
} }
else { else {

View File

@ -30,8 +30,8 @@ import org.apache.hadoop.hbase.util.Bytes;
* See https://issues.apache.org/jira/browse/HBASE-13448 * See https://issues.apache.org/jira/browse/HBASE-13448
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="EQ_DOESNT_OVERRIDE_EQUALS")
public class SizeCachedKeyValue extends KeyValue { public class SizeCachedKeyValue extends KeyValue {
private static final int HEAP_SIZE_OVERHEAD = Bytes.SIZEOF_SHORT + Bytes.SIZEOF_INT; private static final int HEAP_SIZE_OVERHEAD = Bytes.SIZEOF_SHORT + Bytes.SIZEOF_INT;
private short rowLen; private short rowLen;

View File

@ -175,7 +175,7 @@ public class ZKSplitLogManagerCoordination extends ZooKeeperListener implements
return; return;
} }
Task task = findOrCreateOrphanTask(path); Task task = findOrCreateOrphanTask(path);
if (task.isOrphan() && (task.incarnation == 0)) { if (task.isOrphan() && (task.incarnation.get() == 0)) {
LOG.info("resubmitting unassigned orphan task " + path); LOG.info("resubmitting unassigned orphan task " + path);
// ignore failure to resubmit. The timeout-monitor will handle it later // ignore failure to resubmit. The timeout-monitor will handle it later
// albeit in a more crude fashion // albeit in a more crude fashion
@ -228,7 +228,7 @@ public class ZKSplitLogManagerCoordination extends ZooKeeperListener implements
version = -1; version = -1;
} }
LOG.info("resubmitting task " + path); LOG.info("resubmitting task " + path);
task.incarnation++; task.incarnation.incrementAndGet();
boolean result = resubmit(this.details.getServerName(), path, version); boolean result = resubmit(this.details.getServerName(), path, version);
if (!result) { if (!result) {
task.heartbeatNoDetails(EnvironmentEdgeManager.currentTime()); task.heartbeatNoDetails(EnvironmentEdgeManager.currentTime());

View File

@ -78,7 +78,7 @@ public class ZkSplitLogWorkerCoordination extends ZooKeeperListener implements
private TaskExecutor splitTaskExecutor; private TaskExecutor splitTaskExecutor;
private final Object taskReadyLock = new Object(); private final Object taskReadyLock = new Object();
volatile int taskReadySeq = 0; private AtomicInteger taskReadySeq = new AtomicInteger(0);
private volatile String currentTask = null; private volatile String currentTask = null;
private int currentVersion; private int currentVersion;
private volatile boolean shouldStop = false; private volatile boolean shouldStop = false;
@ -106,7 +106,7 @@ public class ZkSplitLogWorkerCoordination extends ZooKeeperListener implements
if (path.equals(watcher.splitLogZNode)) { if (path.equals(watcher.splitLogZNode)) {
if (LOG.isTraceEnabled()) LOG.trace("tasks arrived or departed on " + path); if (LOG.isTraceEnabled()) LOG.trace("tasks arrived or departed on " + path);
synchronized (taskReadyLock) { synchronized (taskReadyLock) {
taskReadySeq++; this.taskReadySeq.incrementAndGet();
taskReadyLock.notify(); taskReadyLock.notify();
} }
} }
@ -400,14 +400,14 @@ public class ZkSplitLogWorkerCoordination extends ZooKeeperListener implements
* policy puts an upper-limit on the number of simultaneous log splitting that could be happening * policy puts an upper-limit on the number of simultaneous log splitting that could be happening
* in a cluster. * in a cluster.
* <p> * <p>
* Synchronization using {@link #taskReadyLock} ensures that it will try to grab every task that * Synchronization using <code>taskReadyLock</code> ensures that it will try to grab every task
* has been put up * that has been put up
* @throws InterruptedException * @throws InterruptedException
*/ */
@Override @Override
public void taskLoop() throws InterruptedException { public void taskLoop() throws InterruptedException {
while (!shouldStop) { while (!shouldStop) {
int seq_start = taskReadySeq; int seq_start = taskReadySeq.get();
List<String> paths = null; List<String> paths = null;
paths = getTaskList(); paths = getTaskList();
if (paths == null) { if (paths == null) {
@ -441,7 +441,7 @@ public class ZkSplitLogWorkerCoordination extends ZooKeeperListener implements
} }
SplitLogCounters.tot_wkr_task_grabing.incrementAndGet(); SplitLogCounters.tot_wkr_task_grabing.incrementAndGet();
synchronized (taskReadyLock) { synchronized (taskReadyLock) {
while (seq_start == taskReadySeq) { while (seq_start == taskReadySeq.get()) {
taskReadyLock.wait(checkInterval); taskReadyLock.wait(checkInterval);
if (server != null) { if (server != null) {
// check to see if we have stale recovering regions in our internal memory state // check to see if we have stale recovering regions in our internal memory state
@ -527,7 +527,7 @@ public class ZkSplitLogWorkerCoordination extends ZooKeeperListener implements
@Override @Override
public int getTaskReadySeq() { public int getTaskReadySeq() {
return taskReadySeq; return taskReadySeq.get();
} }
@Override @Override

View File

@ -149,6 +149,8 @@ public class JMXJsonServlet extends HttpServlet {
* The servlet response we are creating * The servlet response we are creating
*/ */
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="XSS_REQUEST_PARAMETER_TO_SERVLET_WRITER",
justification="TODO: See HBASE-15122")
public void doGet(HttpServletRequest request, HttpServletResponse response) { public void doGet(HttpServletRequest request, HttpServletResponse response) {
try { try {
if (!HttpServer.isInstrumentationAccessAllowed(getServletContext(), request, response)) { if (!HttpServer.isInstrumentationAccessAllowed(getServletContext(), request, response)) {

View File

@ -54,6 +54,8 @@ import org.apache.hadoop.hbase.util.Pair;
* it fallbacks to the archived path. * it fallbacks to the archived path.
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="EQ_DOESNT_OVERRIDE_EQUALS",
justification="To be fixed but warning suppressed for now")
public class HFileLink extends FileLink { public class HFileLink extends FileLink {
private static final Log LOG = LogFactory.getLog(HFileLink.class); private static final Log LOG = LogFactory.getLog(HFileLink.class);

View File

@ -488,6 +488,8 @@ public class HFile {
* @return an appropriate instance of HFileReader * @return an appropriate instance of HFileReader
* @throws IOException If file is invalid, will throw CorruptHFileException flavored IOException * @throws IOException If file is invalid, will throw CorruptHFileException flavored IOException
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="SF_SWITCH_FALLTHROUGH",
justification="Intentional")
private static Reader pickReaderVersion(Path path, FSDataInputStreamWrapper fsdis, private static Reader pickReaderVersion(Path path, FSDataInputStreamWrapper fsdis,
long size, CacheConfig cacheConf, HFileSystem hfs, Configuration conf) throws IOException { long size, CacheConfig cacheConf, HFileSystem hfs, Configuration conf) throws IOException {
FixedFileTrailer trailer = null; FixedFileTrailer trailer = null;
@ -498,7 +500,7 @@ public class HFile {
switch (trailer.getMajorVersion()) { switch (trailer.getMajorVersion()) {
case 2: case 2:
LOG.debug("Opening HFile v2 with v3 reader"); LOG.debug("Opening HFile v2 with v3 reader");
// Fall through. // Fall through. FindBugs: SF_SWITCH_FALLTHROUGH
case 3 : case 3 :
return new HFileReaderImpl(path, trailer, fsdis, size, cacheConf, hfs, conf); return new HFileReaderImpl(path, trailer, fsdis, size, cacheConf, hfs, conf);
default: default:

View File

@ -109,8 +109,8 @@ public class HFileBlock implements Cacheable {
new byte[HConstants.HFILEBLOCK_HEADER_SIZE_NO_CHECKSUM]; new byte[HConstants.HFILEBLOCK_HEADER_SIZE_NO_CHECKSUM];
// How to get the estimate correctly? if it is a singleBB? // How to get the estimate correctly? if it is a singleBB?
public static final int MULTI_BYTE_BUFFER_HEAP_SIZE = (int) ClassSize.estimateBase( public static final int MULTI_BYTE_BUFFER_HEAP_SIZE =
new MultiByteBuff(ByteBuffer.wrap(new byte[0], 0, 0)).getClass(), false); (int)ClassSize.estimateBase(MultiByteBuff.class, false);
// meta.usesHBaseChecksum+offset+nextBlockOnDiskSizeWithHeader // meta.usesHBaseChecksum+offset+nextBlockOnDiskSizeWithHeader
public static final int EXTRA_SERIALIZATION_SPACE = Bytes.SIZEOF_BYTE + Bytes.SIZEOF_INT public static final int EXTRA_SERIALIZATION_SPACE = Bytes.SIZEOF_BYTE + Bytes.SIZEOF_INT

View File

@ -23,6 +23,7 @@ import java.nio.ByteBuffer;
import java.security.Key; import java.security.Key;
import java.util.ArrayList; import java.util.ArrayList;
import java.util.List; import java.util.List;
import java.util.concurrent.atomic.AtomicInteger;
import org.apache.commons.logging.Log; import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory; import org.apache.commons.logging.LogFactory;
@ -473,7 +474,7 @@ public class HFileReaderImpl implements HFile.Reader, Configurable {
private int currMemstoreTSLen; private int currMemstoreTSLen;
private long currMemstoreTS; private long currMemstoreTS;
// Updated but never read? // Updated but never read?
protected volatile int blockFetches; protected AtomicInteger blockFetches = new AtomicInteger(0);
protected final HFile.Reader reader; protected final HFile.Reader reader;
private int currTagsLen; private int currTagsLen;
// buffer backed keyonlyKV // buffer backed keyonlyKV
@ -877,6 +878,8 @@ public class HFileReaderImpl implements HFile.Reader, Configurable {
* @return the next block, or null if there are no more data blocks * @return the next block, or null if there are no more data blocks
* @throws IOException * @throws IOException
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NP_NULL_ON_SOME_PATH",
justification="Yeah, unnecessary null check; could do w/ clean up")
protected HFileBlock readNextDataBlock() throws IOException { protected HFileBlock readNextDataBlock() throws IOException {
long lastDataBlockOffset = reader.getTrailer().getLastDataBlockOffset(); long lastDataBlockOffset = reader.getTrailer().getLastDataBlockOffset();
if (curBlock == null) if (curBlock == null)
@ -885,8 +888,9 @@ public class HFileReaderImpl implements HFile.Reader, Configurable {
HFileBlock block = this.curBlock; HFileBlock block = this.curBlock;
do { do {
if (block.getOffset() >= lastDataBlockOffset) if (block.getOffset() >= lastDataBlockOffset) {
return null; return null;
}
if (block.getOffset() < 0) { if (block.getOffset() < 0) {
throw new IOException("Invalid block file offset: " + block); throw new IOException("Invalid block file offset: " + block);
@ -898,7 +902,7 @@ public class HFileReaderImpl implements HFile.Reader, Configurable {
+ block.getOnDiskSizeWithHeader(), + block.getOnDiskSizeWithHeader(),
block.getNextBlockOnDiskSizeWithHeader(), cacheBlocks, pread, block.getNextBlockOnDiskSizeWithHeader(), cacheBlocks, pread,
isCompaction, true, null, getEffectiveDataBlockEncoding()); isCompaction, true, null, getEffectiveDataBlockEncoding());
if (block != null && !block.getBlockType().isData()) { if (block != null && !block.getBlockType().isData()) { // Findbugs: NP_NULL_ON_SOME_PATH
// Whatever block we read we will be returning it unless // Whatever block we read we will be returning it unless
// it is a datablock. Just in case the blocks are non data blocks // it is a datablock. Just in case the blocks are non data blocks
reader.returnBlock(block); reader.returnBlock(block);
@ -1228,7 +1232,7 @@ public class HFileReaderImpl implements HFile.Reader, Configurable {
updateCurrBlockRef(newBlock); updateCurrBlockRef(newBlock);
blockBuffer = newBlock.getBufferWithoutHeader(); blockBuffer = newBlock.getBufferWithoutHeader();
readKeyValueLen(); readKeyValueLen();
blockFetches++; blockFetches.incrementAndGet();
// Reset the next indexed key // Reset the next indexed key
this.nextIndexedKey = null; this.nextIndexedKey = null;
@ -1667,7 +1671,7 @@ public class HFileReaderImpl implements HFile.Reader, Configurable {
updateCurrBlockRef(newBlock); updateCurrBlockRef(newBlock);
ByteBuff encodedBuffer = getEncodedBuffer(newBlock); ByteBuff encodedBuffer = getEncodedBuffer(newBlock);
seeker.setCurrentBuffer(encodedBuffer); seeker.setCurrentBuffer(encodedBuffer);
blockFetches++; blockFetches.incrementAndGet();
// Reset the next indexed key // Reset the next indexed key
this.nextIndexedKey = null; this.nextIndexedKey = null;

View File

@ -354,7 +354,7 @@ public class HFileWriterImpl implements HFile.Writer {
// (table,startrow,hash) so can't be treated as plain byte arrays. Just skip // (table,startrow,hash) so can't be treated as plain byte arrays. Just skip
// out without // out without
// trying to do this optimization. // trying to do this optimization.
if (comparator != null && comparator instanceof MetaCellComparator) { if (comparator instanceof MetaCellComparator) {
return right; return right;
} }
int diff = comparator.compareRows(left, right); int diff = comparator.compareRows(left, right);

View File

@ -322,6 +322,8 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
private long responseBlockSize = 0; private long responseBlockSize = 0;
private boolean retryImmediatelySupported; private boolean retryImmediatelySupported;
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NP_NULL_ON_SOME_PATH",
justification="Can't figure why this complaint is happening... see below")
Call(int id, final BlockingService service, final MethodDescriptor md, RequestHeader header, Call(int id, final BlockingService service, final MethodDescriptor md, RequestHeader header,
Message param, CellScanner cellScanner, Connection connection, Responder responder, Message param, CellScanner cellScanner, Connection connection, Responder responder,
long size, TraceInfo tinfo, final InetAddress remoteAddress) { long size, TraceInfo tinfo, final InetAddress remoteAddress) {
@ -339,15 +341,18 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
this.isError = false; this.isError = false;
this.size = size; this.size = size;
this.tinfo = tinfo; this.tinfo = tinfo;
this.user = connection == null ? null : connection.user; this.user = connection == null? null: connection.user; // FindBugs: NP_NULL_ON_SOME_PATH
this.remoteAddress = remoteAddress; this.remoteAddress = remoteAddress;
this.retryImmediatelySupported = connection.retryImmediatelySupported; this.retryImmediatelySupported =
connection == null? null: connection.retryImmediatelySupported;
} }
/** /**
* Call is done. Execution happened and we returned results to client. It is now safe to * Call is done. Execution happened and we returned results to client. It is now safe to
* cleanup. * cleanup.
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="Presume the lock on processing request held by caller is protection enough")
void done() { void done() {
if (this.cellBlock != null && reservoir != null) { if (this.cellBlock != null && reservoir != null) {
// Return buffer to reservoir now we are done with it. // Return buffer to reservoir now we are done with it.
@ -599,7 +604,7 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
} }
@Override @Override
public void setCallBack(RpcCallback callback) { public synchronized void setCallBack(RpcCallback callback) {
this.callback = callback; this.callback = callback;
} }
@ -779,6 +784,9 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
} }
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="selector access is not synchronized; seems fine but concerned changing " +
"it will have per impact")
public void run() { public void run() {
LOG.info(getName() + ": starting"); LOG.info(getName() + ": starting");
while (running) { while (running) {
@ -1280,15 +1288,14 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
private boolean useWrap = false; private boolean useWrap = false;
// Fake 'call' for failed authorization response // Fake 'call' for failed authorization response
private static final int AUTHORIZATION_FAILED_CALLID = -1; private static final int AUTHORIZATION_FAILED_CALLID = -1;
private final Call authFailedCall = private final Call authFailedCall = new Call(AUTHORIZATION_FAILED_CALLID, null, null, null,
new Call(AUTHORIZATION_FAILED_CALLID, null, null, null, null, null, this, null, 0, null, null, null, this, null, 0, null, null);
null);
private ByteArrayOutputStream authFailedResponse = private ByteArrayOutputStream authFailedResponse =
new ByteArrayOutputStream(); new ByteArrayOutputStream();
// Fake 'call' for SASL context setup // Fake 'call' for SASL context setup
private static final int SASL_CALLID = -33; private static final int SASL_CALLID = -33;
private final Call saslCall = private final Call saslCall = new Call(SASL_CALLID, null, null, null, null, null, this, null,
new Call(SASL_CALLID, this.service, null, null, null, null, this, null, 0, null, null); 0, null, null);
// was authentication allowed with a fallback to simple auth // was authentication allowed with a fallback to simple auth
private boolean authenticatedWithFallback; private boolean authenticatedWithFallback;
@ -2177,7 +2184,7 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
} }
@Override @Override
public void refreshAuthManager(PolicyProvider pp) { public synchronized void refreshAuthManager(PolicyProvider pp) {
// Ignore warnings that this should be accessed in a static way instead of via an instance; // Ignore warnings that this should be accessed in a static way instead of via an instance;
// it'll break if you go via static route. // it'll break if you go via static route.
this.authManager.refresh(this.conf, pp); this.authManager.refresh(this.conf, pp);
@ -2403,7 +2410,8 @@ public class RpcServer implements RpcServerInterface, ConfigurationObserver {
* @throws org.apache.hadoop.security.authorize.AuthorizationException * @throws org.apache.hadoop.security.authorize.AuthorizationException
* when the client isn't authorized to talk the protocol * when the client isn't authorized to talk the protocol
*/ */
public void authorize(UserGroupInformation user, ConnectionHeader connection, InetAddress addr) public synchronized void authorize(UserGroupInformation user, ConnectionHeader connection,
InetAddress addr)
throws AuthorizationException { throws AuthorizationException {
if (authorize) { if (authorize) {
Class<?> c = getServiceInterface(services, connection.getServiceName()); Class<?> c = getServiceInterface(services, connection.getServiceName());

View File

@ -32,7 +32,6 @@ import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.conf.Configured; import org.apache.hadoop.conf.Configured;
import org.apache.hadoop.fs.FSDataInputStream; import org.apache.hadoop.fs.FSDataInputStream;
import org.apache.hadoop.fs.FSDataOutputStream;
import org.apache.hadoop.fs.FileSystem; import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path; import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hbase.Cell; import org.apache.hadoop.hbase.Cell;
@ -135,17 +134,18 @@ public class HashTable extends Configured implements Tool {
p.setProperty("endTimestamp", Long.toString(endTime)); p.setProperty("endTimestamp", Long.toString(endTime));
} }
FSDataOutputStream out = fs.create(path); try (OutputStreamWriter osw = new OutputStreamWriter(fs.create(path), Charsets.UTF_8)) {
p.store(new OutputStreamWriter(out, Charsets.UTF_8), null); p.store(osw, null);
out.close(); }
} }
void readPropertiesFile(FileSystem fs, Path path) throws IOException { void readPropertiesFile(FileSystem fs, Path path) throws IOException {
FSDataInputStream in = fs.open(path);
Properties p = new Properties(); Properties p = new Properties();
p.load(new InputStreamReader(in, Charsets.UTF_8)); try (FSDataInputStream in = fs.open(path)) {
in.close(); try (InputStreamReader isr = new InputStreamReader(in, Charsets.UTF_8)) {
p.load(isr);
}
}
tableName = p.getProperty("table"); tableName = p.getProperty("table");
families = p.getProperty("columnFamilies"); families = p.getProperty("columnFamilies");
batchSize = Long.parseLong(p.getProperty("targetBatchSize")); batchSize = Long.parseLong(p.getProperty("targetBatchSize"));

View File

@ -142,6 +142,8 @@ public class Import extends Configured implements Tool {
} }
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="EQ_COMPARETO_USE_OBJECT_EQUALS",
justification="This is wrong, yes, but we should be purging Writables, not fixing them")
public int compareTo(KeyValueWritableComparable o) { public int compareTo(KeyValueWritableComparable o) {
return CellComparator.COMPARATOR.compare(this.kv, ((KeyValueWritableComparable)o).kv); return CellComparator.COMPARATOR.compare(this.kv, ((KeyValueWritableComparable)o).kv);
} }
@ -249,6 +251,8 @@ public class Import extends Configured implements Tool {
/** /**
* A mapper that just writes out KeyValues. * A mapper that just writes out KeyValues.
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="EQ_COMPARETO_USE_OBJECT_EQUALS",
justification="Writables are going away and this has been this way forever")
public static class KeyValueImporter extends TableMapper<ImmutableBytesWritable, KeyValue> { public static class KeyValueImporter extends TableMapper<ImmutableBytesWritable, KeyValue> {
private Map<byte[], byte[]> cfRenameMap; private Map<byte[], byte[]> cfRenameMap;
private Filter filter; private Filter filter;

View File

@ -107,7 +107,7 @@ public class ImportTsv extends Configured implements Tool {
* If table didn't exist and was created in dry-run mode, this flag is * If table didn't exist and was created in dry-run mode, this flag is
* flipped to delete it when MR ends. * flipped to delete it when MR ends.
*/ */
private static boolean dryRunTableCreated; private static boolean DRY_RUN_TABLE_CREATED;
public static class TsvParser { public static class TsvParser {
/** /**
@ -475,118 +475,119 @@ public class ImportTsv extends Configured implements Tool {
// See if a non-default Mapper was set // See if a non-default Mapper was set
String mapperClassName = conf.get(MAPPER_CONF_KEY); String mapperClassName = conf.get(MAPPER_CONF_KEY);
Class mapperClass = Class mapperClass = mapperClassName != null? Class.forName(mapperClassName): DEFAULT_MAPPER;
mapperClassName != null ? Class.forName(mapperClassName) : DEFAULT_MAPPER;
TableName tableName = TableName.valueOf(args[0]); TableName tableName = TableName.valueOf(args[0]);
Path inputDir = new Path(args[1]); Path inputDir = new Path(args[1]);
String jobName = conf.get(JOB_NAME_CONF_KEY,NAME + "_" + tableName.getNameAsString()); String jobName = conf.get(JOB_NAME_CONF_KEY,NAME + "_" + tableName.getNameAsString());
job = Job.getInstance(conf, jobName); job = Job.getInstance(conf, jobName);
job.setJarByClass(mapperClass); job.setJarByClass(mapperClass);
FileInputFormat.setInputPaths(job, inputDir); FileInputFormat.setInputPaths(job, inputDir);
job.setInputFormatClass(TextInputFormat.class); job.setInputFormatClass(TextInputFormat.class);
job.setMapperClass(mapperClass); job.setMapperClass(mapperClass);
job.setMapOutputKeyClass(ImmutableBytesWritable.class); job.setMapOutputKeyClass(ImmutableBytesWritable.class);
String hfileOutPath = conf.get(BULK_OUTPUT_CONF_KEY); String hfileOutPath = conf.get(BULK_OUTPUT_CONF_KEY);
String[] columns = conf.getStrings(COLUMNS_CONF_KEY); String[] columns = conf.getStrings(COLUMNS_CONF_KEY);
if(StringUtils.isNotEmpty(conf.get(CREDENTIALS_LOCATION))) { if(StringUtils.isNotEmpty(conf.get(CREDENTIALS_LOCATION))) {
String fileLoc = conf.get(CREDENTIALS_LOCATION); String fileLoc = conf.get(CREDENTIALS_LOCATION);
Credentials cred = Credentials.readTokenStorageFile(new File(fileLoc), conf); Credentials cred = Credentials.readTokenStorageFile(new File(fileLoc), conf);
job.getCredentials().addAll(cred); job.getCredentials().addAll(cred);
} }
if (hfileOutPath != null) { if (hfileOutPath != null) {
if (!admin.tableExists(tableName)) { if (!admin.tableExists(tableName)) {
LOG.warn(format("Table '%s' does not exist.", tableName)); LOG.warn(format("Table '%s' does not exist.", tableName));
if ("yes".equalsIgnoreCase(conf.get(CREATE_TABLE_CONF_KEY, "yes"))) { if ("yes".equalsIgnoreCase(conf.get(CREATE_TABLE_CONF_KEY, "yes"))) {
// TODO: this is backwards. Instead of depending on the existence of a table, // TODO: this is backwards. Instead of depending on the existence of a table,
// create a sane splits file for HFileOutputFormat based on data sampling. // create a sane splits file for HFileOutputFormat based on data sampling.
createTable(admin, tableName, columns); createTable(admin, tableName, columns);
if (isDryRun) { if (isDryRun) {
LOG.warn("Dry run: Table will be deleted at end of dry run."); LOG.warn("Dry run: Table will be deleted at end of dry run.");
dryRunTableCreated = true; synchronized (ImportTsv.class) {
} DRY_RUN_TABLE_CREATED = true;
} else {
String errorMsg =
format("Table '%s' does not exist and '%s' is set to no.", tableName,
CREATE_TABLE_CONF_KEY);
LOG.error(errorMsg);
throw new TableNotFoundException(errorMsg);
}
}
try (Table table = connection.getTable(tableName);
RegionLocator regionLocator = connection.getRegionLocator(tableName)) {
boolean noStrict = conf.getBoolean(NO_STRICT_COL_FAMILY, false);
// if no.strict is false then check column family
if(!noStrict) {
ArrayList<String> unmatchedFamilies = new ArrayList<String>();
Set<String> cfSet = getColumnFamilies(columns);
HTableDescriptor tDesc = table.getTableDescriptor();
for (String cf : cfSet) {
if(tDesc.getFamily(Bytes.toBytes(cf)) == null) {
unmatchedFamilies.add(cf);
}
}
if(unmatchedFamilies.size() > 0) {
ArrayList<String> familyNames = new ArrayList<String>();
for (HColumnDescriptor family : table.getTableDescriptor().getFamilies()) {
familyNames.add(family.getNameAsString());
}
String msg =
"Column Families " + unmatchedFamilies + " specified in " + COLUMNS_CONF_KEY
+ " does not match with any of the table " + tableName
+ " column families " + familyNames + ".\n"
+ "To disable column family check, use -D" + NO_STRICT_COL_FAMILY
+ "=true.\n";
usage(msg);
System.exit(-1);
} }
} }
if (mapperClass.equals(TsvImporterTextMapper.class)) { } else {
job.setMapOutputValueClass(Text.class); String errorMsg =
job.setReducerClass(TextSortReducer.class); format("Table '%s' does not exist and '%s' is set to no.", tableName,
} else { CREATE_TABLE_CONF_KEY);
job.setMapOutputValueClass(Put.class);
job.setCombinerClass(PutCombiner.class);
job.setReducerClass(PutSortReducer.class);
}
if (!isDryRun) {
Path outputDir = new Path(hfileOutPath);
FileOutputFormat.setOutputPath(job, outputDir);
HFileOutputFormat2.configureIncrementalLoad(job, table.getTableDescriptor(),
regionLocator);
}
}
} else {
if (!admin.tableExists(tableName)) {
String errorMsg = format("Table '%s' does not exist.", tableName);
LOG.error(errorMsg); LOG.error(errorMsg);
throw new TableNotFoundException(errorMsg); throw new TableNotFoundException(errorMsg);
} }
}
try (Table table = connection.getTable(tableName);
RegionLocator regionLocator = connection.getRegionLocator(tableName)) {
boolean noStrict = conf.getBoolean(NO_STRICT_COL_FAMILY, false);
// if no.strict is false then check column family
if(!noStrict) {
ArrayList<String> unmatchedFamilies = new ArrayList<String>();
Set<String> cfSet = getColumnFamilies(columns);
HTableDescriptor tDesc = table.getTableDescriptor();
for (String cf : cfSet) {
if(tDesc.getFamily(Bytes.toBytes(cf)) == null) {
unmatchedFamilies.add(cf);
}
}
if(unmatchedFamilies.size() > 0) {
ArrayList<String> familyNames = new ArrayList<String>();
for (HColumnDescriptor family : table.getTableDescriptor().getFamilies()) {
familyNames.add(family.getNameAsString());
}
String msg =
"Column Families " + unmatchedFamilies + " specified in " + COLUMNS_CONF_KEY
+ " does not match with any of the table " + tableName
+ " column families " + familyNames + ".\n"
+ "To disable column family check, use -D" + NO_STRICT_COL_FAMILY
+ "=true.\n";
usage(msg);
System.exit(-1);
}
}
if (mapperClass.equals(TsvImporterTextMapper.class)) { if (mapperClass.equals(TsvImporterTextMapper.class)) {
usage(TsvImporterTextMapper.class.toString() job.setMapOutputValueClass(Text.class);
+ " should not be used for non bulkloading case. use " job.setReducerClass(TextSortReducer.class);
+ TsvImporterMapper.class.toString() } else {
+ " or custom mapper whose value type is Put."); job.setMapOutputValueClass(Put.class);
System.exit(-1); job.setCombinerClass(PutCombiner.class);
job.setReducerClass(PutSortReducer.class);
} }
if (!isDryRun) { if (!isDryRun) {
// No reducers. Just write straight to table. Call initTableReducerJob Path outputDir = new Path(hfileOutPath);
// to set up the TableOutputFormat. FileOutputFormat.setOutputPath(job, outputDir);
TableMapReduceUtil.initTableReducerJob(tableName.getNameAsString(), null, job); HFileOutputFormat2.configureIncrementalLoad(job, table.getTableDescriptor(),
regionLocator);
} }
job.setNumReduceTasks(0);
} }
if (isDryRun) { } else {
job.setOutputFormatClass(NullOutputFormat.class); if (!admin.tableExists(tableName)) {
job.getConfiguration().setStrings("io.serializations", String errorMsg = format("Table '%s' does not exist.", tableName);
job.getConfiguration().get("io.serializations"), LOG.error(errorMsg);
MutationSerialization.class.getName(), ResultSerialization.class.getName(), throw new TableNotFoundException(errorMsg);
KeyValueSerialization.class.getName());
} }
TableMapReduceUtil.addDependencyJars(job); if (mapperClass.equals(TsvImporterTextMapper.class)) {
TableMapReduceUtil.addDependencyJars(job.getConfiguration(), usage(TsvImporterTextMapper.class.toString()
com.google.common.base.Function.class /* Guava used by TsvParser */); + " should not be used for non bulkloading case. use "
+ TsvImporterMapper.class.toString()
+ " or custom mapper whose value type is Put.");
System.exit(-1);
}
if (!isDryRun) {
// No reducers. Just write straight to table. Call initTableReducerJob
// to set up the TableOutputFormat.
TableMapReduceUtil.initTableReducerJob(tableName.getNameAsString(), null, job);
}
job.setNumReduceTasks(0);
}
if (isDryRun) {
job.setOutputFormatClass(NullOutputFormat.class);
job.getConfiguration().setStrings("io.serializations",
job.getConfiguration().get("io.serializations"),
MutationSerialization.class.getName(), ResultSerialization.class.getName(),
KeyValueSerialization.class.getName());
}
TableMapReduceUtil.addDependencyJars(job);
TableMapReduceUtil.addDependencyJars(job.getConfiguration(),
com.google.common.base.Function.class /* Guava used by TsvParser */);
} }
} }
return job; return job;
@ -616,7 +617,8 @@ public class ImportTsv extends Configured implements Tool {
} }
admin.deleteTable(tableName); admin.deleteTable(tableName);
} catch (IOException e) { } catch (IOException e) {
LOG.error(format("***Dry run: Failed to delete table '%s'.***\n%s", tableName, e.toString())); LOG.error(format("***Dry run: Failed to delete table '%s'.***%n%s", tableName,
e.toString()));
return; return;
} }
LOG.info(format("Dry run: Deleted table '%s'.", tableName)); LOG.info(format("Dry run: Deleted table '%s'.", tableName));
@ -658,7 +660,7 @@ public class ImportTsv extends Configured implements Tool {
"input data. Another special column" + TsvParser.TIMESTAMPKEY_COLUMN_SPEC + "input data. Another special column" + TsvParser.TIMESTAMPKEY_COLUMN_SPEC +
" designates that this column should be\n" + " designates that this column should be\n" +
"used as timestamp for each record. Unlike " + TsvParser.ROWKEY_COLUMN_SPEC + ", " + "used as timestamp for each record. Unlike " + TsvParser.ROWKEY_COLUMN_SPEC + ", " +
TsvParser.TIMESTAMPKEY_COLUMN_SPEC + " is optional.\n" + TsvParser.TIMESTAMPKEY_COLUMN_SPEC + " is optional." + "\n" +
"You must specify at most one column as timestamp key for each imported record.\n" + "You must specify at most one column as timestamp key for each imported record.\n" +
"Record with invalid timestamps (blank, non-numeric) will be treated as bad record.\n" + "Record with invalid timestamps (blank, non-numeric) will be treated as bad record.\n" +
"Note: if you use this option, then '" + TIMESTAMP_CONF_KEY + "' option will be ignored.\n" + "Note: if you use this option, then '" + TIMESTAMP_CONF_KEY + "' option will be ignored.\n" +
@ -767,10 +769,16 @@ public class ImportTsv extends Configured implements Tool {
// system time // system time
getConf().setLong(TIMESTAMP_CONF_KEY, timstamp); getConf().setLong(TIMESTAMP_CONF_KEY, timstamp);
dryRunTableCreated = false; synchronized (ImportTsv.class) {
DRY_RUN_TABLE_CREATED = false;
}
Job job = createSubmittableJob(getConf(), args); Job job = createSubmittableJob(getConf(), args);
boolean success = job.waitForCompletion(true); boolean success = job.waitForCompletion(true);
if (dryRunTableCreated) { boolean delete = false;
synchronized (ImportTsv.class) {
delete = DRY_RUN_TABLE_CREATED;
}
if (delete) {
deleteTable(getConf(), args); deleteTable(getConf(), args);
} }
return success ? 0 : 1; return success ? 0 : 1;

View File

@ -110,9 +110,7 @@ public abstract class MultiTableInputFormatBase extends
@Override @Override
public void close() throws IOException { public void close() throws IOException {
trr.close(); trr.close();
if (connection != null) { connection.close();
connection.close();
}
} }
@Override @Override
@ -145,9 +143,7 @@ public abstract class MultiTableInputFormatBase extends
// If there is an exception make sure that all // If there is an exception make sure that all
// resources are closed and released. // resources are closed and released.
trr.close(); trr.close();
if (connection != null) { connection.close();
connection.close();
}
throw ioe; throw ioe;
} }
} }

View File

@ -230,6 +230,8 @@ public class MultithreadedTableMapper<K2, V2> extends TableMapper<K2, V2> {
} }
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION",
justification="Don't understand why FB is complaining about this one. We do throw exception")
private class MapRunner implements Runnable { private class MapRunner implements Runnable {
private Mapper<ImmutableBytesWritable, Result, K2,V2> mapper; private Mapper<ImmutableBytesWritable, Result, K2,V2> mapper;
private Context subcontext; private Context subcontext;
@ -280,7 +282,7 @@ public class MultithreadedTableMapper<K2, V2> extends TableMapper<K2, V2> {
Class<?> wrappedMapperClass = Class.forName("org.apache.hadoop.mapreduce.lib.map.WrappedMapper"); Class<?> wrappedMapperClass = Class.forName("org.apache.hadoop.mapreduce.lib.map.WrappedMapper");
Method getMapContext = wrappedMapperClass.getMethod("getMapContext", MapContext.class); Method getMapContext = wrappedMapperClass.getMethod("getMapContext", MapContext.class);
subcontext = (Context) getMapContext.invoke(wrappedMapperClass.newInstance(), mc); subcontext = (Context) getMapContext.invoke(wrappedMapperClass.newInstance(), mc);
} catch (Exception ee) { } catch (Exception ee) { // FindBugs: REC_CATCH_EXCEPTION
// rethrow as IOE // rethrow as IOE
throw new IOException(e); throw new IOException(e);
} }

View File

@ -111,6 +111,8 @@ implements Configurable {
* org.apache.hadoop.conf.Configuration) * org.apache.hadoop.conf.Configuration)
*/ */
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION",
justification="Intentional")
public void setConf(Configuration configuration) { public void setConf(Configuration configuration) {
this.conf = configuration; this.conf = configuration;

View File

@ -2174,6 +2174,9 @@ public class AssignmentManager {
} }
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(
value="AT_OPERATION_SEQUENCE_ON_CONCURRENT_ABSTRACTION",
justification="Worth fixing but not the end of the world.")
private String onRegionFailedOpen(final RegionState current, private String onRegionFailedOpen(final RegionState current,
final HRegionInfo hri, final ServerName serverName) { final HRegionInfo hri, final ServerName serverName) {
// The region must be opening on this server. // The region must be opening on this server.
@ -2189,6 +2192,7 @@ public class AssignmentManager {
} }
String encodedName = hri.getEncodedName(); String encodedName = hri.getEncodedName();
// FindBugs: AT_OPERATION_SEQUENCE_ON_CONCURRENT_ABSTRACTION Worth fixing!!!
AtomicInteger failedOpenCount = failedOpenTracker.get(encodedName); AtomicInteger failedOpenCount = failedOpenTracker.get(encodedName);
if (failedOpenCount == null) { if (failedOpenCount == null) {
failedOpenCount = new AtomicInteger(); failedOpenCount = new AtomicInteger();

View File

@ -61,6 +61,8 @@ public class ExpiredMobFileCleanerChore extends ScheduledChore {
} }
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION",
justification="Intentional")
protected void chore() { protected void chore() {
try { try {
TableDescriptors htds = master.getTableDescriptors(); TableDescriptors htds = master.getTableDescriptors();

View File

@ -424,8 +424,6 @@ public class RegionStates {
if (!serverName.equals(oldServerName)) { if (!serverName.equals(oldServerName)) {
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug("Onlined " + hri.getShortNameToLog() + " on " + serverName); LOG.debug("Onlined " + hri.getShortNameToLog() + " on " + serverName);
} else {
LOG.debug("Onlined " + hri.getShortNameToLog() + " on " + serverName);
} }
addToServerHoldings(serverName, hri); addToServerHoldings(serverName, hri);
addToReplicaMapping(hri); addToReplicaMapping(hri);

View File

@ -643,7 +643,7 @@ public class SplitLogManager {
public volatile ServerName cur_worker_name; public volatile ServerName cur_worker_name;
public volatile TaskBatch batch; public volatile TaskBatch batch;
public volatile TerminationStatus status; public volatile TerminationStatus status;
public volatile int incarnation; public volatile AtomicInteger incarnation = new AtomicInteger(0);
public final AtomicInteger unforcedResubmits = new AtomicInteger(); public final AtomicInteger unforcedResubmits = new AtomicInteger();
public volatile boolean resubmitThresholdReached; public volatile boolean resubmitThresholdReached;
@ -655,7 +655,6 @@ public class SplitLogManager {
} }
public Task() { public Task() {
incarnation = 0;
last_version = -1; last_version = -1;
status = IN_PROGRESS; status = IN_PROGRESS;
setUnassigned(); setUnassigned();

View File

@ -65,12 +65,15 @@ import com.google.common.collect.Sets;
* is likely to go aways anyways. * is likely to go aways anyways.
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="TODO: synchronize access on nsTable but it is done in tiers above and this " +
"class is going away/shrinking")
public class TableNamespaceManager { public class TableNamespaceManager {
private static final Log LOG = LogFactory.getLog(TableNamespaceManager.class); private static final Log LOG = LogFactory.getLog(TableNamespaceManager.class);
private Configuration conf; private Configuration conf;
private MasterServices masterServices; private MasterServices masterServices;
private Table nsTable = null; private Table nsTable = null; // FindBugs: IS2_INCONSISTENT_SYNC TODO: Access is not synchronized
private ZKNamespaceManager zkNamespaceManager; private ZKNamespaceManager zkNamespaceManager;
private boolean initialized; private boolean initialized;

View File

@ -554,17 +554,21 @@ public abstract class BaseLoadBalancer implements LoadBalancer {
switch (action.type) { switch (action.type) {
case NULL: break; case NULL: break;
case ASSIGN_REGION: case ASSIGN_REGION:
// FindBugs: Having the assert quietens FB BC_UNCONFIRMED_CAST warnings
assert action instanceof AssignRegionAction: action.getClass();
AssignRegionAction ar = (AssignRegionAction) action; AssignRegionAction ar = (AssignRegionAction) action;
regionsPerServer[ar.server] = addRegion(regionsPerServer[ar.server], ar.region); regionsPerServer[ar.server] = addRegion(regionsPerServer[ar.server], ar.region);
regionMoved(ar.region, -1, ar.server); regionMoved(ar.region, -1, ar.server);
break; break;
case MOVE_REGION: case MOVE_REGION:
assert action instanceof MoveRegionAction: action.getClass();
MoveRegionAction mra = (MoveRegionAction) action; MoveRegionAction mra = (MoveRegionAction) action;
regionsPerServer[mra.fromServer] = removeRegion(regionsPerServer[mra.fromServer], mra.region); regionsPerServer[mra.fromServer] = removeRegion(regionsPerServer[mra.fromServer], mra.region);
regionsPerServer[mra.toServer] = addRegion(regionsPerServer[mra.toServer], mra.region); regionsPerServer[mra.toServer] = addRegion(regionsPerServer[mra.toServer], mra.region);
regionMoved(mra.region, mra.fromServer, mra.toServer); regionMoved(mra.region, mra.fromServer, mra.toServer);
break; break;
case SWAP_REGIONS: case SWAP_REGIONS:
assert action instanceof SwapRegionsAction: action.getClass();
SwapRegionsAction a = (SwapRegionsAction) action; SwapRegionsAction a = (SwapRegionsAction) action;
regionsPerServer[a.fromServer] = replaceRegion(regionsPerServer[a.fromServer], a.fromRegion, a.toRegion); regionsPerServer[a.fromServer] = replaceRegion(regionsPerServer[a.fromServer], a.fromRegion, a.toRegion);
regionsPerServer[a.toServer] = replaceRegion(regionsPerServer[a.toServer], a.toRegion, a.fromRegion); regionsPerServer[a.toServer] = replaceRegion(regionsPerServer[a.toServer], a.toRegion, a.fromRegion);
@ -1095,7 +1099,7 @@ public abstract class BaseLoadBalancer implements LoadBalancer {
} }
@Override @Override
public void setClusterStatus(ClusterStatus st) { public synchronized void setClusterStatus(ClusterStatus st) {
this.clusterStatus = st; this.clusterStatus = st;
regionFinder.setClusterStatus(st); regionFinder.setClusterStatus(st);
} }

View File

@ -94,6 +94,8 @@ import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
* so that the balancer gets the full picture of all loads on the cluster.</p> * so that the balancer gets the full picture of all loads on the cluster.</p>
*/ */
@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.CONFIG) @InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.CONFIG)
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="Complaint is about costFunctions not being synchronized; not end of the world")
public class StochasticLoadBalancer extends BaseLoadBalancer { public class StochasticLoadBalancer extends BaseLoadBalancer {
protected static final String STEPS_PER_REGION_KEY = protected static final String STEPS_PER_REGION_KEY =
@ -119,7 +121,7 @@ public class StochasticLoadBalancer extends BaseLoadBalancer {
private CandidateGenerator[] candidateGenerators; private CandidateGenerator[] candidateGenerators;
private CostFromRegionLoadFunction[] regionLoadFunctions; private CostFromRegionLoadFunction[] regionLoadFunctions;
private CostFunction[] costFunctions; private CostFunction[] costFunctions; // FindBugs: Wants this protected; IS2_INCONSISTENT_SYNC
// to save and report costs to JMX // to save and report costs to JMX
private Double curOverallCost = 0d; private Double curOverallCost = 0d;

View File

@ -96,7 +96,7 @@ public class HFileLinkCleaner extends BaseHFileCleanerDelegate {
} }
@Override @Override
public void setConf(Configuration conf) { public synchronized void setConf(Configuration conf) {
super.setConf(conf); super.setConf(conf);
// setup filesystem // setup filesystem

View File

@ -154,6 +154,8 @@ public abstract class TakeSnapshotHandler extends EventHandler implements Snapsh
* call should get implemented for each snapshot flavor. * call should get implemented for each snapshot flavor.
*/ */
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION",
justification="Intentional")
public void process() { public void process() {
String msg = "Running " + snapshot.getType() + " table snapshot " + snapshot.getName() + " " String msg = "Running " + snapshot.getType() + " table snapshot " + snapshot.getName() + " "
+ eventType + " on table " + snapshotTable; + eventType + " on table " + snapshotTable;
@ -205,7 +207,7 @@ public abstract class TakeSnapshotHandler extends EventHandler implements Snapsh
status.markComplete(msg); status.markComplete(msg);
LOG.info(msg); LOG.info(msg);
metricsSnapshot.addSnapshot(status.getCompletionTimestamp() - status.getStartTime()); metricsSnapshot.addSnapshot(status.getCompletionTimestamp() - status.getStartTime());
} catch (Exception e) { } catch (Exception e) { // FindBugs: REC_CATCH_EXCEPTION
status.abort("Failed to complete snapshot " + snapshot.getName() + " on table " + status.abort("Failed to complete snapshot " + snapshot.getName() + " on table " +
snapshotTable + " because " + e.getMessage()); snapshotTable + " because " + e.getMessage());
String reason = "Failed taking snapshot " + ClientSnapshotDescriptionUtils.toString(snapshot) String reason = "Failed taking snapshot " + ClientSnapshotDescriptionUtils.toString(snapshot)

View File

@ -89,6 +89,8 @@ public class ExpiredMobFileCleaner extends Configured implements Tool {
System.err.println(" familyName The column family name"); System.err.println(" familyName The column family name");
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION",
justification="Intentional")
public int run(String[] args) throws Exception { public int run(String[] args) throws Exception {
if (args.length != 2) { if (args.length != 2) {
printUsage(); printUsage();

View File

@ -98,7 +98,7 @@ public class MonitoredRPCHandlerImpl extends MonitoredTaskImpl
* by this Handler. * by this Handler.
* @return a string representing the method call without parameters * @return a string representing the method call without parameters
*/ */
public String getRPC() { public synchronized String getRPC() {
return getRPC(false); return getRPC(false);
} }
@ -166,7 +166,7 @@ public class MonitoredRPCHandlerImpl extends MonitoredTaskImpl
* @return true if the monitored handler is currently servicing an RPC call * @return true if the monitored handler is currently servicing an RPC call
* to a database command. * to a database command.
*/ */
public boolean isOperationRunning() { public synchronized boolean isOperationRunning() {
if(!isRPCRunning()) { if(!isRPCRunning()) {
return false; return false;
} }
@ -212,7 +212,7 @@ public class MonitoredRPCHandlerImpl extends MonitoredTaskImpl
} }
@Override @Override
public void markComplete(String status) { public synchronized void markComplete(String status) {
super.markComplete(status); super.markComplete(status);
this.params = null; this.params = null;
this.packet = null; this.packet = null;

View File

@ -28,6 +28,9 @@ import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@InterfaceStability.Evolving @InterfaceStability.Evolving
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="FindBugs seems confused; says globalLimiter and lastUpdate " +
"are mostly synchronized...but to me it looks like they are totally synchronized")
public class QuotaState { public class QuotaState {
protected long lastUpdate = 0; protected long lastUpdate = 0;
protected long lastQuery = 0; protected long lastQuery = 0;
@ -77,7 +80,7 @@ public class QuotaState {
* Setup the global quota information. * Setup the global quota information.
* (This operation is part of the QuotaState setup) * (This operation is part of the QuotaState setup)
*/ */
public void setQuotas(final Quotas quotas) { public synchronized void setQuotas(final Quotas quotas) {
if (quotas.hasThrottle()) { if (quotas.hasThrottle()) {
globalLimiter = QuotaLimiterFactory.fromThrottle(quotas.getThrottle()); globalLimiter = QuotaLimiterFactory.fromThrottle(quotas.getThrottle());
} else { } else {

View File

@ -45,6 +45,9 @@ import com.google.common.annotations.VisibleForTesting;
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@InterfaceStability.Evolving @InterfaceStability.Evolving
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="FindBugs seems confused; says limit and tlimit " +
"are mostly synchronized...but to me it looks like they are totally synchronized")
public abstract class RateLimiter { public abstract class RateLimiter {
public static final String QUOTA_RATE_LIMITER_CONF_KEY = "hbase.quota.rate.limiter"; public static final String QUOTA_RATE_LIMITER_CONF_KEY = "hbase.quota.rate.limiter";
private long tunit = 1000; // Timeunit factor for translating to ms. private long tunit = 1000; // Timeunit factor for translating to ms.
@ -73,7 +76,7 @@ public abstract class RateLimiter {
* @param limit The max value available resource units can be refilled to. * @param limit The max value available resource units can be refilled to.
* @param timeUnit Timeunit factor for translating to ms. * @param timeUnit Timeunit factor for translating to ms.
*/ */
public void set(final long limit, final TimeUnit timeUnit) { public synchronized void set(final long limit, final TimeUnit timeUnit) {
switch (timeUnit) { switch (timeUnit) {
case MILLISECONDS: case MILLISECONDS:
tunit = 1; tunit = 1;
@ -99,10 +102,11 @@ public abstract class RateLimiter {
public String toString() { public String toString() {
String rateLimiter = this.getClass().getSimpleName(); String rateLimiter = this.getClass().getSimpleName();
if (limit == Long.MAX_VALUE) { if (getLimit() == Long.MAX_VALUE) {
return rateLimiter + "(Bypass)"; return rateLimiter + "(Bypass)";
} }
return rateLimiter + "(avail=" + avail + " limit=" + limit + " tunit=" + tunit + ")"; return rateLimiter + "(avail=" + getAvailable() + " limit=" + getLimit() +
" tunit=" + getTimeUnitInMillis() + ")";
} }
/** /**
@ -120,7 +124,7 @@ public abstract class RateLimiter {
} }
public synchronized boolean isBypass() { public synchronized boolean isBypass() {
return limit == Long.MAX_VALUE; return getLimit() == Long.MAX_VALUE;
} }
public synchronized long getLimit() { public synchronized long getLimit() {
@ -131,7 +135,7 @@ public abstract class RateLimiter {
return avail; return avail;
} }
protected long getTimeUnitInMillis() { protected synchronized long getTimeUnitInMillis() {
return tunit; return tunit;
} }
@ -195,7 +199,7 @@ public abstract class RateLimiter {
*/ */
public synchronized long waitInterval(final long amount) { public synchronized long waitInterval(final long amount) {
// TODO Handle over quota? // TODO Handle over quota?
return (amount <= avail) ? 0 : getWaitInterval(limit, avail, amount); return (amount <= avail) ? 0 : getWaitInterval(getLimit(), avail, amount);
} }
// These two method are for strictly testing purpose only // These two method are for strictly testing purpose only

View File

@ -34,6 +34,9 @@ import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@InterfaceStability.Evolving @InterfaceStability.Evolving
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="FindBugs seems confused; says bypassGlobals, namepaceLimiters, and " +
"tableLimiters are mostly synchronized...but to me it looks like they are totally synchronized")
public class UserQuotaState extends QuotaState { public class UserQuotaState extends QuotaState {
private Map<String, QuotaLimiter> namespaceLimiters = null; private Map<String, QuotaLimiter> namespaceLimiters = null;
private Map<TableName, QuotaLimiter> tableLimiters = null; private Map<TableName, QuotaLimiter> tableLimiters = null;
@ -96,7 +99,7 @@ public class UserQuotaState extends QuotaState {
} }
@Override @Override
public void setQuotas(final Quotas quotas) { public synchronized void setQuotas(final Quotas quotas) {
super.setQuotas(quotas); super.setQuotas(quotas);
bypassGlobals = quotas.getBypassGlobals(); bypassGlobals = quotas.getBypassGlobals();
} }
@ -105,7 +108,7 @@ public class UserQuotaState extends QuotaState {
* Add the quota information of the specified table. * Add the quota information of the specified table.
* (This operation is part of the QuotaState setup) * (This operation is part of the QuotaState setup)
*/ */
public void setQuotas(final TableName table, Quotas quotas) { public synchronized void setQuotas(final TableName table, Quotas quotas) {
tableLimiters = setLimiter(tableLimiters, table, quotas); tableLimiters = setLimiter(tableLimiters, table, quotas);
} }

View File

@ -393,7 +393,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
// Set when a flush has been requested. // Set when a flush has been requested.
volatile boolean flushRequested = false; volatile boolean flushRequested = false;
// Number of compactions running. // Number of compactions running.
volatile int compacting = 0; AtomicInteger compacting = new AtomicInteger(0);
// Gets set in close. If set, cannot compact or flush again. // Gets set in close. If set, cannot compact or flush again.
volatile boolean writesEnabled = true; volatile boolean writesEnabled = true;
// Set if region is read-only // Set if region is read-only
@ -824,7 +824,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
this.writestate.setReadOnly(ServerRegionReplicaUtil.isReadOnly(this)); this.writestate.setReadOnly(ServerRegionReplicaUtil.isReadOnly(this));
this.writestate.flushRequested = false; this.writestate.flushRequested = false;
this.writestate.compacting = 0; this.writestate.compacting.set(0);
if (this.writestate.writesEnabled) { if (this.writestate.writesEnabled) {
// Remove temporary data left over from old regions // Remove temporary data left over from old regions
@ -1368,6 +1368,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
this.closing.set(closing); this.closing.set(closing);
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="UL_UNRELEASED_LOCK_EXCEPTION_PATH",
justification="I think FindBugs is confused")
private Map<byte[], List<StoreFile>> doClose(final boolean abort, MonitoredTask status) private Map<byte[], List<StoreFile>> doClose(final boolean abort, MonitoredTask status)
throws IOException { throws IOException {
if (isClosed()) { if (isClosed()) {
@ -1405,7 +1407,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
// block waiting for the lock for closing // block waiting for the lock for closing
lock.writeLock().lock(); lock.writeLock().lock(); // FindBugs: Complains UL_UNRELEASED_LOCK_EXCEPTION_PATH but seems fine
this.closing.set(true); this.closing.set(true);
status.setStatus("Disabling writes for close"); status.setStatus("Disabling writes for close");
try { try {
@ -1537,7 +1539,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
boolean interrupted = false; boolean interrupted = false;
try { try {
while (writestate.compacting > 0 || writestate.flushing) { while (writestate.compacting.get() > 0 || writestate.flushing) {
LOG.debug("waiting for " + writestate.compacting + " compactions" LOG.debug("waiting for " + writestate.compacting + " compactions"
+ (writestate.flushing ? " & cache flush" : "") + " to complete for region " + this); + (writestate.flushing ? " & cache flush" : "") + " to complete for region " + this);
try { try {
@ -1894,7 +1896,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
synchronized (writestate) { synchronized (writestate) {
if (writestate.writesEnabled) { if (writestate.writesEnabled) {
wasStateSet = true; wasStateSet = true;
++writestate.compacting; writestate.compacting.incrementAndGet();
} else { } else {
String msg = "NOT compacting region " + this + ". Writes disabled."; String msg = "NOT compacting region " + this + ". Writes disabled.";
LOG.info(msg); LOG.info(msg);
@ -1920,8 +1922,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} finally { } finally {
if (wasStateSet) { if (wasStateSet) {
synchronized (writestate) { synchronized (writestate) {
--writestate.compacting; writestate.compacting.decrementAndGet();
if (writestate.compacting <= 0) { if (writestate.compacting.get() <= 0) {
writestate.notifyAll(); writestate.notifyAll();
} }
} }
@ -2164,6 +2166,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="DLS_DEAD_LOCAL_STORE",
justification="FindBugs seems confused about trxId")
protected PrepareFlushResult internalPrepareFlushCache(final WAL wal, final long myseqid, protected PrepareFlushResult internalPrepareFlushCache(final WAL wal, final long myseqid,
final Collection<Store> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker) final Collection<Store> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker)
throws IOException { throws IOException {
@ -2395,6 +2399,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
return false; return false;
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NN_NAKED_NOTIFY",
justification="Intentional; notify is about completed flush")
protected FlushResult internalFlushCacheAndCommit( protected FlushResult internalFlushCacheAndCommit(
final WAL wal, MonitoredTask status, final PrepareFlushResult prepareResult, final WAL wal, MonitoredTask status, final PrepareFlushResult prepareResult,
final Collection<Store> storesToFlush) final Collection<Store> storesToFlush)
@ -4448,6 +4454,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
@VisibleForTesting @VisibleForTesting
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NN_NAKED_NOTIFY",
justification="Intentional; post memstore flush")
void replayWALFlushCommitMarker(FlushDescriptor flush) throws IOException { void replayWALFlushCommitMarker(FlushDescriptor flush) throws IOException {
MonitoredTask status = TaskMonitor.get().createStatus("Committing flush " + this); MonitoredTask status = TaskMonitor.get().createStatus("Committing flush " + this);
@ -4684,6 +4692,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
return prepareFlushResult; return prepareFlushResult;
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NN_NAKED_NOTIFY",
justification="Intentional; cleared the memstore")
void replayWALRegionEventMarker(RegionEventDescriptor regionEvent) throws IOException { void replayWALRegionEventMarker(RegionEventDescriptor regionEvent) throws IOException {
checkTargetRegion(regionEvent.getEncodedRegionName().toByteArray(), checkTargetRegion(regionEvent.getEncodedRegionName().toByteArray(),
"RegionEvent marker from WAL ", regionEvent); "RegionEvent marker from WAL ", regionEvent);
@ -4914,6 +4924,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
return refreshStoreFiles(false); return refreshStoreFiles(false);
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NN_NAKED_NOTIFY",
justification="Notify is about post replay. Intentional")
protected boolean refreshStoreFiles(boolean force) throws IOException { protected boolean refreshStoreFiles(boolean force) throws IOException {
if (!force && ServerRegionReplicaUtil.isDefaultReplica(this.getRegionInfo())) { if (!force && ServerRegionReplicaUtil.isDefaultReplica(this.getRegionInfo())) {
return false; // if primary nothing to do return false; // if primary nothing to do
@ -7838,6 +7850,8 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
} }
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="SF_SWITCH_FALLTHROUGH",
justification="Intentional")
public void startRegionOperation(Operation op) throws IOException { public void startRegionOperation(Operation op) throws IOException {
switch (op) { switch (op) {
case GET: // read operations case GET: // read operations

View File

@ -189,7 +189,7 @@ implements HeapSize, Map<K,V> {
* *
* @return currently available bytes * @return currently available bytes
*/ */
public long getMemFree() { public synchronized long getMemFree() {
return memFree; return memFree;
} }
@ -208,7 +208,7 @@ implements HeapSize, Map<K,V> {
* @return currently used memory in bytes * @return currently used memory in bytes
*/ */
public long getMemUsed() { public long getMemUsed() {
return (memTotal - memFree); // FindBugs IS2_INCONSISTENT_SYNC return (memTotal - getMemFree()); // FindBugs IS2_INCONSISTENT_SYNC
} }
/** /**
@ -227,7 +227,7 @@ implements HeapSize, Map<K,V> {
* *
* @return number of misses * @return number of misses
*/ */
public long getMissCount() { public synchronized long getMissCount() {
return missCount; // FindBugs IS2_INCONSISTENT_SYNC return missCount; // FindBugs IS2_INCONSISTENT_SYNC
} }
@ -239,7 +239,7 @@ implements HeapSize, Map<K,V> {
*/ */
public double getHitRatio() { public double getHitRatio() {
return (double)((double)hitCount/ return (double)((double)hitCount/
((double)(hitCount+missCount))); ((double)(hitCount + getMissCount())));
} }
/** /**
@ -269,7 +269,7 @@ implements HeapSize, Map<K,V> {
* @return memory usage of map in bytes * @return memory usage of map in bytes
*/ */
public long heapSize() { public long heapSize() {
return (memTotal - memFree); return (memTotal - getMemFree());
} }
//-------------------------------------------------------------------------- //--------------------------------------------------------------------------
@ -824,6 +824,8 @@ implements HeapSize, Map<K,V> {
* *
* @return Set of entries in hash * @return Set of entries in hash
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
justification="Unused debugging function that reads only")
public Set<Entry<K,V>> entryTableSet() { public Set<Entry<K,V>> entryTableSet() {
Set<Entry<K,V>> entrySet = new HashSet<Entry<K,V>>(); Set<Entry<K,V>> entrySet = new HashSet<Entry<K,V>>();
Entry [] table = entries; // FindBugs IS2_INCONSISTENT_SYNC Entry [] table = entries; // FindBugs IS2_INCONSISTENT_SYNC

View File

@ -60,7 +60,7 @@ public class MemStoreChunkPool {
final static float POOL_INITIAL_SIZE_DEFAULT = 0.0f; final static float POOL_INITIAL_SIZE_DEFAULT = 0.0f;
// Static reference to the MemStoreChunkPool // Static reference to the MemStoreChunkPool
private static MemStoreChunkPool globalInstance; private static MemStoreChunkPool GLOBAL_INSTANCE;
/** Boolean whether we have disabled the memstore chunk pool entirely. */ /** Boolean whether we have disabled the memstore chunk pool entirely. */
static boolean chunkPoolDisabled = false; static boolean chunkPoolDisabled = false;
@ -179,12 +179,14 @@ public class MemStoreChunkPool {
* @param conf * @param conf
* @return the global MemStoreChunkPool instance * @return the global MemStoreChunkPool instance
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="DC_DOUBLECHECK",
justification="Intentional")
static MemStoreChunkPool getPool(Configuration conf) { static MemStoreChunkPool getPool(Configuration conf) {
if (globalInstance != null) return globalInstance; if (GLOBAL_INSTANCE != null) return GLOBAL_INSTANCE;
synchronized (MemStoreChunkPool.class) { synchronized (MemStoreChunkPool.class) {
if (chunkPoolDisabled) return null; if (chunkPoolDisabled) return null;
if (globalInstance != null) return globalInstance; if (GLOBAL_INSTANCE != null) return GLOBAL_INSTANCE;
float poolSizePercentage = conf.getFloat(CHUNK_POOL_MAXSIZE_KEY, POOL_MAX_SIZE_DEFAULT); float poolSizePercentage = conf.getFloat(CHUNK_POOL_MAXSIZE_KEY, POOL_MAX_SIZE_DEFAULT);
if (poolSizePercentage <= 0) { if (poolSizePercentage <= 0) {
chunkPoolDisabled = true; chunkPoolDisabled = true;
@ -210,8 +212,8 @@ public class MemStoreChunkPool {
int initialCount = (int) (initialCountPercentage * maxCount); int initialCount = (int) (initialCountPercentage * maxCount);
LOG.info("Allocating MemStoreChunkPool with chunk size " + StringUtils.byteDesc(chunkSize) LOG.info("Allocating MemStoreChunkPool with chunk size " + StringUtils.byteDesc(chunkSize)
+ ", max count " + maxCount + ", initial count " + initialCount); + ", max count " + maxCount + ", initial count " + initialCount);
globalInstance = new MemStoreChunkPool(conf, chunkSize, maxCount, initialCount); GLOBAL_INSTANCE = new MemStoreChunkPool(conf, chunkSize, maxCount, initialCount);
return globalInstance; return GLOBAL_INSTANCE;
} }
} }

View File

@ -278,6 +278,8 @@ public class RegionServerCoprocessorHost extends
private RegionServerServices regionServerServices; private RegionServerServices regionServerServices;
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="BC_UNCONFIRMED_CAST",
justification="Intentional; FB has trouble detecting isAssignableFrom")
public RegionServerEnvironment(final Class<?> implClass, public RegionServerEnvironment(final Class<?> implClass,
final Coprocessor impl, final int priority, final int seq, final Coprocessor impl, final int priority, final int seq,
final Configuration conf, final RegionServerServices services) { final Configuration conf, final RegionServerServices services) {
@ -285,7 +287,7 @@ public class RegionServerCoprocessorHost extends
this.regionServerServices = services; this.regionServerServices = services;
for (Object itf : ClassUtils.getAllInterfaces(implClass)) { for (Object itf : ClassUtils.getAllInterfaces(implClass)) {
Class<?> c = (Class<?>) itf; Class<?> c = (Class<?>) itf;
if (SingletonCoprocessorService.class.isAssignableFrom(c)) { if (SingletonCoprocessorService.class.isAssignableFrom(c)) {// FindBugs: BC_UNCONFIRMED_CAST
this.regionServerServices.registerService( this.regionServerServices.registerService(
((SingletonCoprocessorService) impl).getService()); ((SingletonCoprocessorService) impl).getService());
break; break;

View File

@ -30,7 +30,6 @@ import java.util.SortedSet;
import java.util.UUID; import java.util.UUID;
import java.util.concurrent.atomic.AtomicBoolean; import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicInteger; import java.util.concurrent.atomic.AtomicInteger;
import java.util.concurrent.locks.ReentrantReadWriteLock;
import org.apache.commons.logging.Log; import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory; import org.apache.commons.logging.LogFactory;
@ -38,11 +37,11 @@ import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileSystem; import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path; import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hbase.Cell; import org.apache.hadoop.hbase.Cell;
import org.apache.hadoop.hbase.CellComparator;
import org.apache.hadoop.hbase.CellUtil; import org.apache.hadoop.hbase.CellUtil;
import org.apache.hadoop.hbase.HConstants; import org.apache.hadoop.hbase.HConstants;
import org.apache.hadoop.hbase.HDFSBlocksDistribution; import org.apache.hadoop.hbase.HDFSBlocksDistribution;
import org.apache.hadoop.hbase.KeyValue; import org.apache.hadoop.hbase.KeyValue;
import org.apache.hadoop.hbase.CellComparator;
import org.apache.hadoop.hbase.KeyValueUtil; import org.apache.hadoop.hbase.KeyValueUtil;
import org.apache.hadoop.hbase.classification.InterfaceAudience; import org.apache.hadoop.hbase.classification.InterfaceAudience;
import org.apache.hadoop.hbase.client.Scan; import org.apache.hadoop.hbase.client.Scan;
@ -51,6 +50,7 @@ import org.apache.hadoop.hbase.io.TimeRange;
import org.apache.hadoop.hbase.io.hfile.BlockType; import org.apache.hadoop.hbase.io.hfile.BlockType;
import org.apache.hadoop.hbase.io.hfile.CacheConfig; import org.apache.hadoop.hbase.io.hfile.CacheConfig;
import org.apache.hadoop.hbase.io.hfile.HFile; import org.apache.hadoop.hbase.io.hfile.HFile;
import org.apache.hadoop.hbase.io.hfile.HFileBlock;
import org.apache.hadoop.hbase.io.hfile.HFileContext; import org.apache.hadoop.hbase.io.hfile.HFileContext;
import org.apache.hadoop.hbase.io.hfile.HFileScanner; import org.apache.hadoop.hbase.io.hfile.HFileScanner;
import org.apache.hadoop.hbase.nio.ByteBuff; import org.apache.hadoop.hbase.nio.ByteBuff;
@ -61,7 +61,6 @@ import org.apache.hadoop.hbase.util.BloomFilterWriter;
import org.apache.hadoop.hbase.util.Bytes; import org.apache.hadoop.hbase.util.Bytes;
import org.apache.hadoop.hbase.util.Writables; import org.apache.hadoop.hbase.util.Writables;
import org.apache.hadoop.io.WritableUtils; import org.apache.hadoop.io.WritableUtils;
import org.apache.hadoop.hbase.io.hfile.HFileBlock;
import com.google.common.annotations.VisibleForTesting; import com.google.common.annotations.VisibleForTesting;
import com.google.common.base.Function; import com.google.common.base.Function;
@ -623,6 +622,8 @@ public class StoreFile {
return false; return false;
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="ICAST_INTEGER_MULTIPLY_CAST_TO_LONG",
justification="Will not overflow")
public static class WriterBuilder { public static class WriterBuilder {
private final Configuration conf; private final Configuration conf;
private final CacheConfig cacheConf; private final CacheConfig cacheConf;
@ -635,7 +636,6 @@ public class StoreFile {
private Path filePath; private Path filePath;
private InetSocketAddress[] favoredNodes; private InetSocketAddress[] favoredNodes;
private HFileContext fileContext; private HFileContext fileContext;
private boolean shouldDropCacheBehind = false;
public WriterBuilder(Configuration conf, CacheConfig cacheConf, public WriterBuilder(Configuration conf, CacheConfig cacheConf,
FileSystem fs) { FileSystem fs) {
@ -703,8 +703,8 @@ public class StoreFile {
return this; return this;
} }
public WriterBuilder withShouldDropCacheBehind(boolean shouldDropCacheBehind) { public WriterBuilder withShouldDropCacheBehind(boolean shouldDropCacheBehind/*NOT USED!!*/) {
this.shouldDropCacheBehind = shouldDropCacheBehind; // TODO: HAS NO EFFECT!!! FIX!!
return this; return this;
} }
/** /**
@ -807,9 +807,6 @@ public class StoreFile {
private Cell lastDeleteFamilyCell = null; private Cell lastDeleteFamilyCell = null;
private long deleteFamilyCnt = 0; private long deleteFamilyCnt = 0;
/** Bytes per Checksum */
protected int bytesPerChecksum;
TimeRangeTracker timeRangeTracker = new TimeRangeTracker(); TimeRangeTracker timeRangeTracker = new TimeRangeTracker();
/* isTimeRangeTrackerSet keeps track if the timeRange has already been set /* isTimeRangeTrackerSet keeps track if the timeRange has already been set
* When flushing a memstore, we set TimeRange and use this variable to * When flushing a memstore, we set TimeRange and use this variable to

View File

@ -91,6 +91,8 @@ public class TimeRangeTracker implements Writable {
* If required, update the current TimestampRange to include timestamp * If required, update the current TimestampRange to include timestamp
* @param timestamp the timestamp value to include * @param timestamp the timestamp value to include
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="MT_CORRECTNESS",
justification="Intentional")
void includeTimestamp(final long timestamp) { void includeTimestamp(final long timestamp) {
// Do test outside of synchronization block. Synchronization in here can be problematic // Do test outside of synchronization block. Synchronization in here can be problematic
// when many threads writing one Store -- they can all pile up trying to add in here. // when many threads writing one Store -- they can all pile up trying to add in here.

View File

@ -332,7 +332,7 @@ public abstract class Compactor {
throughputController.start(compactionName); throughputController.start(compactionName);
KeyValueScanner kvs = (scanner instanceof KeyValueScanner)? (KeyValueScanner)scanner : null; KeyValueScanner kvs = (scanner instanceof KeyValueScanner)? (KeyValueScanner)scanner : null;
int minFilesToCompact = Math.max(2, long minFilesToCompact = Math.max(2L,
conf.getInt(CompactionConfiguration.HBASE_HSTORE_COMPACTION_MIN_KEY, conf.getInt(CompactionConfiguration.HBASE_HSTORE_COMPACTION_MIN_KEY,
/* old name */ conf.getInt("hbase.hstore.compactionThreshold", 3))); /* old name */ conf.getInt("hbase.hstore.compactionThreshold", 3)));
long shippedCallSizeLimit = (long) minFilesToCompact * HConstants.DEFAULT_BLOCKSIZE; long shippedCallSizeLimit = (long) minFilesToCompact * HConstants.DEFAULT_BLOCKSIZE;

View File

@ -43,18 +43,20 @@ import org.apache.zookeeper.KeeperException.SessionExpiredException;
* target cluster is an HBase cluster. * target cluster is an HBase cluster.
*/ */
@InterfaceAudience.Private @InterfaceAudience.Private
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="MT_CORRECTNESS",
justification="Thinks zkw needs to be synchronized access but should be fine as is.")
public abstract class HBaseReplicationEndpoint extends BaseReplicationEndpoint public abstract class HBaseReplicationEndpoint extends BaseReplicationEndpoint
implements Abortable { implements Abortable {
private static final Log LOG = LogFactory.getLog(HBaseReplicationEndpoint.class); private static final Log LOG = LogFactory.getLog(HBaseReplicationEndpoint.class);
private ZooKeeperWatcher zkw = null; private ZooKeeperWatcher zkw = null; // FindBugs: MT_CORRECTNESS
private List<ServerName> regionServers = new ArrayList<ServerName>(0); private List<ServerName> regionServers = new ArrayList<ServerName>(0);
private volatile long lastRegionServerUpdate; private long lastRegionServerUpdate;
protected void disconnect() { protected void disconnect() {
if (zkw != null){ if (zkw != null) {
zkw.close(); zkw.close();
} }
} }
@ -181,7 +183,7 @@ public abstract class HBaseReplicationEndpoint extends BaseReplicationEndpoint
* Set the list of region servers for that peer * Set the list of region servers for that peer
* @param regionServers list of addresses for the region servers * @param regionServers list of addresses for the region servers
*/ */
public void setRegionServers(List<ServerName> regionServers) { public synchronized void setRegionServers(List<ServerName> regionServers) {
this.regionServers = regionServers; this.regionServers = regionServers;
lastRegionServerUpdate = System.currentTimeMillis(); lastRegionServerUpdate = System.currentTimeMillis();
} }

View File

@ -115,7 +115,7 @@ public class RegionReplicaReplicationEndpoint extends HBaseReplicationEndpoint {
* Skips the entries which has original seqId. Only entries persisted via distributed log replay * Skips the entries which has original seqId. Only entries persisted via distributed log replay
* have their original seq Id fields set. * have their original seq Id fields set.
*/ */
private class SkipReplayedEditsFilter extends BaseWALEntryFilter { private static class SkipReplayedEditsFilter extends BaseWALEntryFilter {
@Override @Override
public Entry filter(Entry entry) { public Entry filter(Entry entry) {
// if orig seq id is set, skip replaying the entry // if orig seq id is set, skip replaying the entry

View File

@ -280,7 +280,6 @@ public class SecureBulkLoadEndpoint extends SecureBulkLoadService
Configuration conf = env.getConfiguration(); Configuration conf = env.getConfiguration();
fs = FileSystem.get(conf); fs = FileSystem.get(conf);
for(Pair<byte[], String> el: familyPaths) { for(Pair<byte[], String> el: familyPaths) {
Path p = new Path(el.getSecond());
Path stageFamily = new Path(bulkToken, Bytes.toString(el.getFirst())); Path stageFamily = new Path(bulkToken, Bytes.toString(el.getFirst()));
if(!fs.exists(stageFamily)) { if(!fs.exists(stageFamily)) {
fs.mkdirs(stageFamily); fs.mkdirs(stageFamily);

View File

@ -336,12 +336,6 @@ public class TableAuthManager implements Closeable {
return false; return false;
} }
private boolean authorize(List<TablePermission> perms,
TableName table, byte[] family,
Permission.Action action) {
return authorize(perms, table, family, null, action);
}
private boolean authorize(List<TablePermission> perms, private boolean authorize(List<TablePermission> perms,
TableName table, byte[] family, TableName table, byte[] family,
byte[] qualifier, Permission.Action action) { byte[] qualifier, Permission.Action action) {

View File

@ -124,7 +124,7 @@ public class AuthenticationTokenSecretManager
} }
@Override @Override
protected byte[] createPassword(AuthenticationTokenIdentifier identifier) { protected synchronized byte[] createPassword(AuthenticationTokenIdentifier identifier) {
long now = EnvironmentEdgeManager.currentTime(); long now = EnvironmentEdgeManager.currentTime();
AuthenticationKey secretKey = currentKey; AuthenticationKey secretKey = currentKey;
identifier.setKeyId(secretKey.getKeyId()); identifier.setKeyId(secretKey.getKeyId());
@ -229,7 +229,7 @@ public class AuthenticationTokenSecretManager
return true; return true;
} }
AuthenticationKey getCurrentKey() { synchronized AuthenticationKey getCurrentKey() {
return currentKey; return currentKey;
} }
@ -338,8 +338,11 @@ public class AuthenticationTokenSecretManager
// clear any expired // clear any expired
removeExpiredKeys(); removeExpiredKeys();
long localLastKeyUpdate;
if (lastKeyUpdate + keyUpdateInterval < now) { synchronized (this) {
localLastKeyUpdate = lastKeyUpdate;
}
if (localLastKeyUpdate + keyUpdateInterval < now) {
// roll a new master key // roll a new master key
rollCurrentKey(); rollCurrentKey();
} }

View File

@ -274,8 +274,10 @@ public class VisibilityController extends BaseMasterAndRegionObserver implements
// Read the entire labels table and populate the zk // Read the entire labels table and populate the zk
if (e.getEnvironment().getRegion().getRegionInfo().getTable().equals(LABELS_TABLE_NAME)) { if (e.getEnvironment().getRegion().getRegionInfo().getTable().equals(LABELS_TABLE_NAME)) {
this.labelsRegion = true; this.labelsRegion = true;
this.accessControllerAvailable = CoprocessorHost.getLoadedCoprocessors() synchronized (this) {
this.accessControllerAvailable = CoprocessorHost.getLoadedCoprocessors()
.contains(AccessController.class.getName()); .contains(AccessController.class.getName());
}
// Defer the init of VisibilityLabelService on labels region until it is in recovering state. // Defer the init of VisibilityLabelService on labels region until it is in recovering state.
if (!e.getEnvironment().getRegion().isRecovering()) { if (!e.getEnvironment().getRegion().isRecovering()) {
initVisibilityLabelService(e.getEnvironment()); initVisibilityLabelService(e.getEnvironment());

View File

@ -643,7 +643,6 @@ public class ExportSnapshot extends Configured implements Tool {
@Override @Override
public List<InputSplit> getSplits(JobContext context) throws IOException, InterruptedException { public List<InputSplit> getSplits(JobContext context) throws IOException, InterruptedException {
Configuration conf = context.getConfiguration(); Configuration conf = context.getConfiguration();
String snapshotName = conf.get(CONF_SNAPSHOT_NAME);
Path snapshotDir = new Path(conf.get(CONF_SNAPSHOT_DIR)); Path snapshotDir = new Path(conf.get(CONF_SNAPSHOT_DIR));
FileSystem fs = FileSystem.get(snapshotDir.toUri(), conf); FileSystem fs = FileSystem.get(snapshotDir.toUri(), conf);

View File

@ -280,6 +280,8 @@ public final class SnapshotInfo extends Configured implements Tool {
private SnapshotManifest snapshotManifest; private SnapshotManifest snapshotManifest;
@Override @Override
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="REC_CATCH_EXCEPTION",
justification="Intentional")
public int run(String[] args) throws IOException, InterruptedException { public int run(String[] args) throws IOException, InterruptedException {
final Configuration conf = getConf(); final Configuration conf = getConf();
boolean listSnapshots = false; boolean listSnapshots = false;
@ -317,7 +319,7 @@ public final class SnapshotInfo extends Configured implements Tool {
printUsageAndExit(); printUsageAndExit();
} }
} catch (Exception e) { } catch (Exception e) {
printUsageAndExit(); printUsageAndExit(); // FindBugs: REC_CATCH_EXCEPTION
} }
} }

View File

@ -3564,9 +3564,8 @@ public class HBaseFsck extends Configured implements Closeable {
this.metaEntry = metaEntry; this.metaEntry = metaEntry;
} }
public int getReplicaId() { public synchronized int getReplicaId() {
if (metaEntry != null) return metaEntry.getReplicaId(); return metaEntry != null? metaEntry.getReplicaId(): deployedReplicaId;
return deployedReplicaId;
} }
public synchronized void addServer(HRegionInfo hri, ServerName server) { public synchronized void addServer(HRegionInfo hri, ServerName server) {

View File

@ -68,12 +68,17 @@ public class IdReadWriteLock {
/** For testing */ /** For testing */
@VisibleForTesting @VisibleForTesting
int purgeAndGetEntryPoolSize() { int purgeAndGetEntryPoolSize() {
System.gc(); gc();
Threads.sleep(200); Threads.sleep(200);
lockPool.purge(); lockPool.purge();
return lockPool.size(); return lockPool.size();
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="DM_GC", justification="Intentional")
private void gc() {
System.gc();
}
@VisibleForTesting @VisibleForTesting
public void waitForWaiters(long id, int numWaiters) throws InterruptedException { public void waitForWaiters(long id, int numWaiters) throws InterruptedException {
for (ReentrantReadWriteLock readWriteLock;;) { for (ReentrantReadWriteLock readWriteLock;;) {

View File

@ -104,11 +104,8 @@ public class MetaUtils {
* @return HRegion for meta region * @return HRegion for meta region
* @throws IOException e * @throws IOException e
*/ */
public HRegion getMetaRegion() throws IOException { public synchronized HRegion getMetaRegion() throws IOException {
if (this.metaRegion == null) { return this.metaRegion == null? openMetaRegion(): this.metaRegion;
openMetaRegion();
}
return this.metaRegion;
} }
/** /**

View File

@ -455,9 +455,11 @@ public class RegionMover extends AbstractHBaseTool {
} }
} }
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="DLS_DEAD_LOCAL_STORE",
justification="FB is wrong; its size is read")
private void unloadRegions(Admin admin, String server, ArrayList<String> regionServers, private void unloadRegions(Admin admin, String server, ArrayList<String> regionServers,
boolean ack, List<HRegionInfo> movedRegions) throws Exception { boolean ack, List<HRegionInfo> movedRegions) throws Exception {
List<HRegionInfo> regionsToMove = new ArrayList<HRegionInfo>(); List<HRegionInfo> regionsToMove = new ArrayList<HRegionInfo>();// FindBugs: DLS_DEAD_LOCAL_STORE
regionsToMove = getRegions(this.conf, server); regionsToMove = getRegions(this.conf, server);
if (regionsToMove.size() == 0) { if (regionsToMove.size() == 0) {
LOG.info("No Regions to move....Quitting now"); LOG.info("No Regions to move....Quitting now");
@ -597,7 +599,7 @@ public class RegionMover extends AbstractHBaseTool {
* Move Regions without Acknowledging.Usefule in case of RS shutdown as we might want to shut the * Move Regions without Acknowledging.Usefule in case of RS shutdown as we might want to shut the
* RS down anyways and not abort on a stuck region. Improves movement performance * RS down anyways and not abort on a stuck region. Improves movement performance
*/ */
private class MoveWithoutAck implements Callable<Boolean> { private static class MoveWithoutAck implements Callable<Boolean> {
private Admin admin; private Admin admin;
private HRegionInfo region; private HRegionInfo region;
private String targetServer; private String targetServer;
@ -764,7 +766,7 @@ public class RegionMover extends AbstractHBaseTool {
try { try {
br = new BufferedReader(new FileReader(f)); br = new BufferedReader(new FileReader(f));
while ((line = br.readLine()) != null) { while ((line = br.readLine()) != null) {
line.trim(); line = line.trim();
if (!line.equals("")) { if (!line.equals("")) {
excludeServers.add(line); excludeServers.add(line);
} }

View File

@ -46,6 +46,8 @@ import java.util.RandomAccess;
* <p> * <p>
* Iterators are read-only. They cannot be used to remove elements. * Iterators are read-only. They cannot be used to remove elements.
*/ */
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="UG_SYNC_SET_UNSYNC_GET",
justification="TODO: synchronization in here needs review!!!")
public class SortedList<E> implements List<E>, RandomAccess { public class SortedList<E> implements List<E>, RandomAccess {
private volatile List<E> list; private volatile List<E> list;
private final Comparator<? super E> comparator; private final Comparator<? super E> comparator;
@ -80,7 +82,7 @@ public class SortedList<E> implements List<E>, RandomAccess {
* method to get a reference for iterating over using the RandomAccess * method to get a reference for iterating over using the RandomAccess
* pattern. * pattern.
*/ */
public List<E> get() { public List<E> get() { // FindBugs: UG_SYNC_SET_UNSYNC_GET complaint. Fix!!
return list; return list;
} }
@ -185,7 +187,7 @@ public class SortedList<E> implements List<E>, RandomAccess {
} }
@Override @Override
public E get(int index) { public synchronized E get(int index) {
return list.get(index); return list.get(index);
} }

View File

@ -345,7 +345,7 @@ public class TestSplitLogManager {
assertTrue(task == task2); assertTrue(task == task2);
LOG.debug("task = " + task); LOG.debug("task = " + task);
assertEquals(1L, tot_mgr_resubmit.get()); assertEquals(1L, tot_mgr_resubmit.get());
assertEquals(1, task.incarnation); assertEquals(1, task.incarnation.get());
assertEquals(0, task.unforcedResubmits.get()); assertEquals(0, task.unforcedResubmits.get());
assertTrue(task.isOrphan()); assertTrue(task.isOrphan());
assertTrue(task.isUnassigned()); assertTrue(task.isUnassigned());