Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -56,7 +56,7 @@
* to be true, metrics will be emitted into the FSNamesystem metrics registry
* for each operation which acquires this lock indicating how long the operation
* held the lock for. These metrics have names of the form
* FSN(Read|Write)LockNanosOperationName, where OperationName denotes the name
* ${LockName}(Read|Write)LockNanosOperationName, where OperationName denotes the name
* of the operation that initiated the lock hold (this will be OTHER for certain
* uncategorized operations) and they export the hold time values in
* nanoseconds. Note that if a thread dies, metrics produced after the
Expand All @@ -67,6 +67,7 @@
public class FSNamesystemLock {
@VisibleForTesting
protected ReentrantReadWriteLock coarseLock;
private final String lockName;

private volatile boolean metricsEnabled;
private final MutableRatesWithAggregation detailedHoldTimeMetrics;
Expand Down Expand Up @@ -123,23 +124,26 @@ public Long initialValue() {

@VisibleForTesting
static final String OP_NAME_OTHER = "OTHER";
private static final String READ_LOCK_METRIC_PREFIX = "FSNReadLock";
private static final String WRITE_LOCK_METRIC_PREFIX = "FSNWriteLock";
private final String readLockMetricPrefix;
private final String writeLockMetricPrefix;
private static final String LOCK_METRIC_SUFFIX = "Nanos";

private static final String OVERALL_METRIC_NAME = "Overall";

public FSNamesystemLock(Configuration conf,
public FSNamesystemLock(Configuration conf, String lockName,
MutableRatesWithAggregation detailedHoldTimeMetrics) {
this(conf, detailedHoldTimeMetrics, new Timer());
this(conf, lockName, detailedHoldTimeMetrics, new Timer());
}

@VisibleForTesting
FSNamesystemLock(Configuration conf,
FSNamesystemLock(Configuration conf, String lockName,
MutableRatesWithAggregation detailedHoldTimeMetrics, Timer timer) {
this.lockName = lockName;
this.readLockMetricPrefix = this.lockName + "ReadLock";
this.writeLockMetricPrefix = this.lockName + "WriteLock";
boolean fair = conf.getBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY,
DFS_NAMENODE_FSLOCK_FAIR_DEFAULT);
FSNamesystem.LOG.info("fsLock is fair: " + fair);
FSNamesystem.LOG.info("{}Lock is fair: {}.", this.lockName, fair);
this.coarseLock = new ReentrantReadWriteLock(fair);
this.timer = timer;

Expand All @@ -157,8 +161,8 @@ public FSNamesystemLock(Configuration conf,
this.metricsEnabled = conf.getBoolean(
DFS_NAMENODE_LOCK_DETAILED_METRICS_KEY,
DFS_NAMENODE_LOCK_DETAILED_METRICS_DEFAULT);
FSNamesystem.LOG.info("Detailed lock hold time metrics enabled: " +
this.metricsEnabled);
FSNamesystem.LOG.info("Detailed lock hold time metrics of {}Lock is {}.",
this.lockName, this.metricsEnabled ? "enabled" : "disabled");
this.detailedHoldTimeMetrics = detailedHoldTimeMetrics;
}

Expand Down Expand Up @@ -231,9 +235,9 @@ public void readUnlock(String opName,
LockHeldInfo lockHeldInfo =
longestReadLockHeldInfo.getAndSet(new LockHeldInfo());
FSNamesystem.LOG.info(
"\tNumber of suppressed read-lock reports: {}"
"\tNumber of suppressed read-lock reports of {}Lock is {}"
+ "\n\tLongest read-lock held at {} for {}ms by {}{} via {}",
numSuppressedWarnings, Time.formatTime(lockHeldInfo.getStartTimeMs()),
this.lockName, numSuppressedWarnings, Time.formatTime(lockHeldInfo.getStartTimeMs()),
lockHeldInfo.getIntervalMs(), lockHeldInfo.getOpName(),
lockHeldInfo.getLockReportInfo(), lockHeldInfo.getStackTrace());
}
Expand Down Expand Up @@ -337,10 +341,10 @@ private void writeUnlock(String opName, boolean suppressWriteLockReport,

if (logAction.shouldLog()) {
FSNamesystem.LOG.info(
"\tNumber of suppressed write-lock reports: {}"
"\tNumber of suppressed write-lock reports of {}Lock is {}"
+ "\n\tLongest write-lock held at {} for {}ms by {}{} via {}"
+ "\n\tTotal suppressed write-lock held time: {}",
logAction.getCount() - 1,
this.lockName, logAction.getCount() - 1,
Time.formatTime(lockHeldInfo.getStartTimeMs()),
lockHeldInfo.getIntervalMs(), lockHeldInfo.getOpName(),
lockHeldInfo.getLockReportInfo(), lockHeldInfo.getStackTrace(),
Expand Down Expand Up @@ -456,8 +460,8 @@ private static void updateProcessingDetails(Timing type, long deltaNanos) {
}
}

private static String getMetricName(String operationName, boolean isWrite) {
return (isWrite ? WRITE_LOCK_METRIC_PREFIX : READ_LOCK_METRIC_PREFIX) +
private String getMetricName(String operationName, boolean isWrite) {
return (isWrite ? this.writeLockMetricPrefix : this.readLockMetricPrefix) +
org.apache.commons.lang3.StringUtils.capitalize(operationName) +
LOCK_METRIC_SUFFIX;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -35,8 +35,8 @@ public class FineGrainedFSNamesystemLock implements FSNLockManager {
private final FSNamesystemLock bmLock;

public FineGrainedFSNamesystemLock(Configuration conf, MutableRatesWithAggregation aggregation) {
this.fsLock = new FSNamesystemLock(conf, aggregation);
this.bmLock = new FSNamesystemLock(conf, aggregation);
this.fsLock = new FSNamesystemLock(conf, "FS", aggregation);
this.bmLock = new FSNamesystemLock(conf, "BM", aggregation);
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -29,7 +29,7 @@ public class GlobalFSNamesystemLock implements FSNLockManager {
private final FSNamesystemLock lock;

public GlobalFSNamesystemLock(Configuration conf, MutableRatesWithAggregation aggregation) {
this.lock = new FSNamesystemLock(conf, aggregation);
this.lock = new FSNamesystemLock(conf, "FSN", aggregation);
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -57,17 +57,17 @@ public void testFsLockFairness() throws IOException, InterruptedException{
Configuration conf = new Configuration();

conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, true);
FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null);
FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null);
assertTrue(fsnLock.coarseLock.isFair());

conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, false);
fsnLock = new FSNamesystemLock(conf, null);
fsnLock = new FSNamesystemLock(conf, "FSN", null);
assertFalse(fsnLock.coarseLock.isFair());
}

@Test
public void testFSNamesystemLockCompatibility() {
FSNamesystemLock rwLock = new FSNamesystemLock(new Configuration(), null);
FSNamesystemLock rwLock = new FSNamesystemLock(new Configuration(), "FSN", null);

assertEquals(0, rwLock.getReadHoldCount());
rwLock.readLock();
Expand Down Expand Up @@ -107,7 +107,7 @@ public void testFSLockGetWaiterCount() throws InterruptedException {
final CountDownLatch latch = new CountDownLatch(threadCount);
final Configuration conf = new Configuration();
conf.setBoolean(DFS_NAMENODE_FSLOCK_FAIR_KEY, true);
final FSNamesystemLock rwLock = new FSNamesystemLock(conf, null);
final FSNamesystemLock rwLock = new FSNamesystemLock(conf, "FSN", null);
rwLock.writeLock();
ExecutorService helper = Executors.newFixedThreadPool(threadCount);

Expand Down Expand Up @@ -150,7 +150,7 @@ public void testFSWriteLockLongHoldingReport() throws Exception {
writeLockSuppressWarningInterval, TimeUnit.MILLISECONDS);

final FakeTimer timer = new FakeTimer();
final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null, timer);
final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null, timer);
timer.advance(writeLockSuppressWarningInterval);

LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
Expand Down Expand Up @@ -213,7 +213,7 @@ public void testFSWriteLockLongHoldingReport() throws Exception {
"held at " + Time.formatTime(timer.now()).substring(0, 10);
assertTrue(logs.getOutput().contains(startTimeStr));
assertTrue(logs.getOutput().contains(
"Number of suppressed write-lock reports: 2"));
"Number of suppressed write-lock reports of FSNLock is 2"));
}

/**
Expand All @@ -233,7 +233,7 @@ public void testFSReadLockLongHoldingReport() throws Exception {
readLockSuppressWarningInterval, TimeUnit.MILLISECONDS);

final FakeTimer timer = new FakeTimer();
final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null, timer);
final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null, timer);
timer.advance(readLockSuppressWarningInterval);

LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
Expand Down Expand Up @@ -304,7 +304,7 @@ public void run() {
"held at " + Time.formatTime(timer.now()).substring(0, 10);
assertTrue(logs.getOutput().contains(startTimeStr));
assertTrue(logs.getOutput().contains(
"Number of suppressed read-lock reports: 3"));
"Number of suppressed read-lock reports of FSNLock is 3"));

// Report if it's held for a long time (and time since last report
// exceeds the suppress warning interval) while another thread also has the
Expand Down Expand Up @@ -366,7 +366,7 @@ public void testDetailedHoldMetrics() throws Exception {
MetricsRegistry registry = new MetricsRegistry("Test");
MutableRatesWithAggregation rates =
registry.newRatesWithAggregation("Test");
FSNamesystemLock fsLock = new FSNamesystemLock(conf, rates, timer);
FSNamesystemLock fsLock = new FSNamesystemLock(conf, "FSN", rates, timer);

fsLock.readLock();
timer.advanceNanos(1300000);
Expand Down Expand Up @@ -419,7 +419,7 @@ public void testFSWriteLockReportSuppressed() throws Exception {
writeLockSuppressWarningInterval, TimeUnit.MILLISECONDS);

final FakeTimer timer = new FakeTimer();
final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, null, timer);
final FSNamesystemLock fsnLock = new FSNamesystemLock(conf, "FSN", null, timer);
timer.advance(writeLockSuppressWarningInterval);

LogCapturer logs = LogCapturer.captureLogs(FSNamesystem.LOG);
Expand Down