-
Notifications
You must be signed in to change notification settings - Fork 3.3k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
HBASE-27968 add JvmPauseMonitor in hbase-client #5319
base: master
Are you sure you want to change the base?
Changes from all commits
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -23,14 +23,14 @@ | |
import java.util.Map; | ||
import java.util.Set; | ||
import java.util.concurrent.TimeUnit; | ||
import java.util.concurrent.atomic.AtomicInteger; | ||
import org.apache.hadoop.conf.Configuration; | ||
import org.apache.hadoop.hbase.metrics.JvmPauseMonitorSource; | ||
import org.apache.yetus.audience.InterfaceAudience; | ||
import org.slf4j.Logger; | ||
import org.slf4j.LoggerFactory; | ||
|
||
import org.apache.hbase.thirdparty.com.google.common.base.Joiner; | ||
import org.apache.hbase.thirdparty.com.google.common.base.Preconditions; | ||
import org.apache.hbase.thirdparty.com.google.common.base.Stopwatch; | ||
import org.apache.hbase.thirdparty.com.google.common.collect.Lists; | ||
import org.apache.hbase.thirdparty.com.google.common.collect.Maps; | ||
|
@@ -46,9 +46,13 @@ | |
* which detects GC pauses(Todd Lipcon) | ||
*/ | ||
@InterfaceAudience.Private | ||
public class JvmPauseMonitor { | ||
public final class JvmPauseMonitor { | ||
private static final Logger LOG = LoggerFactory.getLogger(JvmPauseMonitor.class); | ||
|
||
private static final AtomicInteger REF_CNT = new AtomicInteger(); | ||
|
||
private static JvmPauseMonitor INSTANCE; | ||
|
||
/** The target sleep time */ | ||
private static final long SLEEP_INTERVAL_MS = 500; | ||
|
||
|
@@ -62,28 +66,47 @@ public class JvmPauseMonitor { | |
public static final String INFO_THRESHOLD_KEY = "jvm.pause.info-threshold.ms"; | ||
private static final long INFO_THRESHOLD_DEFAULT = 1000; | ||
|
||
public static final String PAUSE_MONITOR_ENABLE_KEY = "hbase.client.pause.monitor.enable"; | ||
|
||
public static final boolean PAUSE_MONITOR_ENABLE_DEFAULT = false; | ||
|
||
private Thread monitorThread; | ||
private volatile boolean shouldRun = true; | ||
private JvmPauseMonitorSource metricsSource; | ||
private final JvmPauseMonitorSource metricsSource; | ||
|
||
public static synchronized JvmPauseMonitor getInstance(Configuration conf) { | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Why change it to singleton? What if we pass different Configuration here? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. There are two reasons. @wchevreuil mentioned that our user may open multiple connections, however only one pause monitor is enough for a process. More details please see here There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. If we only want one JvmPauseMonitor in process, then we should not bind it to a connection? Just add a utility method to enable jvm pause monitor is enough? There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. Ok. Let me try to fix this. Thanks Duo ! @Apache9 |
||
return getInstance(conf, null); | ||
} | ||
|
||
public JvmPauseMonitor(Configuration conf) { | ||
this(conf, null); | ||
public static synchronized JvmPauseMonitor getInstance(Configuration conf, | ||
JvmPauseMonitorSource metricsSource) { | ||
if (INSTANCE == null) { | ||
INSTANCE = new JvmPauseMonitor(conf, metricsSource); | ||
} | ||
return INSTANCE; | ||
} | ||
|
||
public JvmPauseMonitor(Configuration conf, JvmPauseMonitorSource metricsSource) { | ||
private JvmPauseMonitor(Configuration conf, JvmPauseMonitorSource metricsSource) { | ||
this.warnThresholdMs = conf.getLong(WARN_THRESHOLD_KEY, WARN_THRESHOLD_DEFAULT); | ||
this.infoThresholdMs = conf.getLong(INFO_THRESHOLD_KEY, INFO_THRESHOLD_DEFAULT); | ||
this.metricsSource = metricsSource; | ||
} | ||
|
||
public void start() { | ||
Preconditions.checkState(monitorThread == null, "Already started"); | ||
public synchronized void start() { | ||
if (REF_CNT.getAndIncrement() > 0) { | ||
// pause monitor already started | ||
return; | ||
} | ||
monitorThread = new Thread(new Monitor(), "JvmPauseMonitor"); | ||
monitorThread.setDaemon(true); | ||
monitorThread.start(); | ||
} | ||
|
||
public void stop() { | ||
public synchronized void stop() { | ||
if (REF_CNT.decrementAndGet() > 0) { | ||
// there are still open connections | ||
return; | ||
} | ||
shouldRun = false; | ||
monitorThread.interrupt(); | ||
try { | ||
|
@@ -113,7 +136,7 @@ private Map<String, GcTimes> getGcTimes() { | |
return map; | ||
} | ||
|
||
private static class GcTimes { | ||
private static final class GcTimes { | ||
private GcTimes(GarbageCollectorMXBean gcBean) { | ||
gcCount = gcBean.getCollectionCount(); | ||
gcTimeMillis = gcBean.getCollectionTime(); | ||
|
@@ -196,17 +219,13 @@ public JvmPauseMonitorSource getMetricsSource() { | |
return metricsSource; | ||
} | ||
|
||
public void setMetricsSource(JvmPauseMonitorSource metricsSource) { | ||
this.metricsSource = metricsSource; | ||
} | ||
|
||
/** | ||
* Simple 'main' to facilitate manual testing of the pause monitor. This main function just leaks | ||
* memory into a list. Running this class with a 1GB heap will very quickly go into "GC hell" and | ||
* result in log messages about the GC pauses. | ||
*/ | ||
public static void main(String[] args) throws Exception { | ||
new JvmPauseMonitor(new Configuration()).start(); | ||
JvmPauseMonitor.getInstance(new Configuration()).start(); | ||
List<String> list = Lists.newArrayList(); | ||
int i = 0; | ||
while (true) { | ||
|
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Make this a singleton, instead? In the case of applications opening or pooling multiple connections, we don't need a separate JvmPauseMonitor per connection, just a single one for the whole application.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Good idea ! But if so, we may need to consider adding a reference count to pauseMonitor. Only when all connections that refer to it are no longer active, we can stop it . Or we can keep it all the time , even all the connections are closed ?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Yeah, I think we don't need to stop it when connection is closed, it may be useful to have it running for as long the client application runs.