yashmayya commented on code in PR #17851:
URL: https://github.com/apache/pinot/pull/17851#discussion_r2917947910
##########
pinot-core/src/main/java/org/apache/pinot/core/util/trace/ContinuousJfrStarter.java:
##########
@@ -101,22 +98,23 @@ public class ContinuousJfrStarter implements
PinotClusterConfigChangeListener {
///
/// This is only used if toDisk is true.
/// The value is in bytes.
- /// The default value is 200MB.
+ /// The default value is 2G. Valid values are human readable data size
strings, as defined by the DataSizeUtils class
+ /// (e.g. '10.5G', '40B') or data size in bytes (e.g. '2147483648').
///
/// @see #MAX_AGE
public static final String MAX_SIZE = "maxSize";
- public static final int DEFAULT_MAX_SIZE = 200 * 1024 * 1024;
+ public static final String DEFAULT_MAX_SIZE = "2GB";
/// Key that controls the maximum age of the recording file.
/// Once the file reaches this age, older events will be discarded.
/// If both maxSize and maxAge are set, the recording will be discarded when
either condition is met.
///
/// This is only used if toDisk is true.
/// The value is a duration string, as defined by the Duration class.
- /// The default value is 1 day (P1D).
+ /// The default value is 7 days (P7D).
Review Comment:
Curious - why these larger defaults?
##########
pinot-core/src/test/java/org/apache/pinot/core/util/trace/ContinuousJfrStarterTest.java:
##########
@@ -157,66 +154,194 @@ public void noOpWhenNewConfigIsEqual() {
Assertions.assertThat(_continuousJfrStarter.isRunning())
.describedAs("Recording should still be disabled")
.isFalse();
- Mockito.verifyNoInteractions(_recording);
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).isEmpty();
+ }
+
+ @Test
+ public void restartsWhenConfigurationChanges() {
+ Map<String, String> config = Map.of(
+ "pinot.jfr.enabled", "true",
+ "pinot.jfr.maxAge", "PT2H");
+ _continuousJfrStarter.onChange(Set.of(), config);
+
+ Set<String> changed = Set.of("pinot.jfr.maxAge");
+ Map<String, String> updatedConfig = Map.of(
+ "pinot.jfr.enabled", "true",
+ "pinot.jfr.maxAge", "PT1H");
+ _continuousJfrStarter.onChange(changed, updatedConfig);
+
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).containsExactly(
+ "jfrStart name=pinot-continuous settings=default dumponexit=true
disk=true maxsize=" + DEFAULT_MAX_SIZE_BYTES
+ + " maxage=7200000ms",
+ "jfrStop name=pinot-continuous",
+ "jfrStart name=pinot-continuous settings=default dumponexit=true
disk=true maxsize=" + DEFAULT_MAX_SIZE_BYTES
+ + " maxage=3600000ms");
+ }
+
+ @Test
+ public void configuresRepositoryAndDumpPathsViaMBean() {
+ Map<String, String> config = Map.of(
+ "pinot.jfr.enabled", "true",
+ "pinot.jfr.directory", "/var/log/pinot/jfr-repository",
+ "pinot.jfr.dumpPath", "/var/log/pinot/jfr-dumps");
+ _continuousJfrStarter.onChange(Set.of(), config);
+
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).containsExactly(
+ "jfrConfigure repositorypath=/var/log/pinot/jfr-repository
dumppath=/var/log/pinot/jfr-dumps",
+ "jfrStart name=pinot-continuous settings=default dumponexit=true
disk=true maxsize=" + DEFAULT_MAX_SIZE_BYTES
+ + " maxage=" + DEFAULT_MAX_AGE_MILLIS + "ms");
+ }
+
+ @Test
+ public void doesNotFailWhenMBeanIsUnavailable() {
+ _continuousJfrStarter.setMBeanAvailable(false);
+ Map<String, String> config = Map.of("pinot.jfr.enabled", "true");
+ _continuousJfrStarter.onChange(Set.of(), config);
+
+ Assertions.assertThat(_continuousJfrStarter.isRunning())
+ .describedAs("Recording should remain disabled when MBean is
unavailable")
+ .isFalse();
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).isEmpty();
+ }
+
+ @Test
+ public void keepsRunningWhenStopCommandFails() {
+ _continuousJfrStarter.onChange(Set.of(), Map.of("pinot.jfr.enabled",
"true"));
+ _continuousJfrStarter.failCommand("jfrStop");
+
+ _continuousJfrStarter.onChange(Set.of("pinot.jfr.enabled"),
Map.of("pinot.jfr.enabled", "false"));
+
+ Assertions.assertThat(_continuousJfrStarter.isRunning())
+ .describedAs("Starter should keep running state when stop fails")
+ .isTrue();
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).containsExactly(
+ "jfrStart name=pinot-continuous settings=default dumponexit=true
disk=true maxsize=" + DEFAULT_MAX_SIZE_BYTES
+ + " maxage=" + DEFAULT_MAX_AGE_MILLIS + "ms",
+ "jfrStop name=pinot-continuous");
}
@Test
- public void cleanUpThreadDeletesFiles()
- throws IOException {
- Path tempDirectory = Files.createTempDirectory("jfr-test-");
- int maxDumps = 3;
- long now = ZonedDateTime.of(2025, 10, 13, 12, 0, 0, 0,
ZoneOffset.UTC).toInstant().toEpochMilli();
+ public void keepsRunningWhenMBeanBecomesUnavailableOnStop() {
+ _continuousJfrStarter.onChange(Set.of(), Map.of("pinot.jfr.enabled",
"true"));
+ _continuousJfrStarter.setMBeanAvailable(false);
+
+ _continuousJfrStarter.onChange(Set.of("pinot.jfr.enabled"),
Map.of("pinot.jfr.enabled", "false"));
+
+ Assertions.assertThat(_continuousJfrStarter.isRunning())
+ .describedAs("Starter should keep running state when MBean is
unavailable for stop")
+ .isTrue();
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).containsExactly(
+ "jfrStart name=pinot-continuous settings=default dumponexit=true
disk=true maxsize=" + DEFAULT_MAX_SIZE_BYTES
+ + " maxage=" + DEFAULT_MAX_AGE_MILLIS + "ms");
+ }
+
+ @Test
+ public void supportsHumanReadableMaxSize() {
+ Map<String, String> config = Map.of(
+ "pinot.jfr.enabled", "true",
+ "pinot.jfr.maxSize", "512MB");
+ _continuousJfrStarter.onChange(Set.of(), config);
+
+
Assertions.assertThat(_continuousJfrStarter.getExecutedCommands()).containsExactly(
+ "jfrStart name=pinot-continuous settings=default dumponexit=true
disk=true maxsize=536870912 maxage="
+ + DEFAULT_MAX_AGE_MILLIS + "ms");
+ }
+
+ @Test
+ public void integrationTestWithRealDiagnosticCommandMBean() {
+ ContinuousJfrStarter starter = new ContinuousJfrStarter();
+ if (!starter.isDiagnosticCommandAvailable()) {
+ throw new SkipException("JFR DiagnosticCommand MBean is not available in
this runtime");
+ }
+
+ String recordingName = "pinot-continuous-it-" + System.currentTimeMillis();
+ Map<String, String> enabledConfig = Map.of(
+ "pinot.jfr.enabled", "true",
+ "pinot.jfr.name", recordingName,
+ "pinot.jfr.toDisk", "false");
+ Map<String, String> disabledConfig = Map.of(
+ "pinot.jfr.enabled", "false",
+ "pinot.jfr.name", recordingName,
+ "pinot.jfr.toDisk", "false");
+
try {
- long[] dates = IntStream.range(0, maxDumps * 2)
- .mapToLong(i -> now - i * 3600_000L)
- .sorted()
- .toArray();
- for (long creationDate : dates) {
- Path path = ContinuousJfrStarter.getRecordingPath(tempDirectory,
"test", Instant.ofEpochMilli(creationDate));
- File file = path.toFile();
- Assertions.assertThat(file.createNewFile())
- .describedAs("Should be able to create a file in the temp
directory")
- .isTrue();
- Assertions.assertThat(file.setLastModified(creationDate))
- .describedAs("Should be able to set the last modified time")
- .isTrue();
- }
+ starter.onChange(Set.of(), enabledConfig);
+ Assertions.assertThat(starter.isRunning()).isTrue();
+ Assertions.assertThat(waitForRecordingPresence(recordingName, true, 10,
100))
+ .describedAs("Recording should be visible via JFR.check")
+ .isTrue();
- // Verify that we have 2 * maxDumps files
- try (var files = Files.list(tempDirectory)) {
- Assertions.assertThat(files.count())
- .describedAs("Should have 2 * maxDumps files in the temp
directory")
- .isEqualTo(maxDumps * 2);
+ starter.onChange(Set.of("pinot.jfr.enabled"), disabledConfig);
+ Assertions.assertThat(starter.isRunning()).isFalse();
+ Assertions.assertThat(waitForRecordingPresence(recordingName, false, 10,
100))
+ .describedAs("Recording should be absent via JFR.check after
disable")
+ .isTrue();
+ } finally {
+ if (starter.isRunning()) {
+ starter.onChange(Set.of("pinot.jfr.enabled"), disabledConfig);
}
+ }
+ }
- // Run the cleanup
- ContinuousJfrStarter.cleanUpDumps(tempDirectory, maxDumps, "test");
-
- // Verify that we have maxDumps files and only the newest ones are kept
- try (var files = Files.list(tempDirectory)) {
- var remainingFiles = files.collect(Collectors.toSet());
- Assertions.assertThat(remainingFiles)
- .describedAs("Should have maxDumps files in the temp directory")
- .hasSize(maxDumps);
- for (int i = 0; i < maxDumps; i++) {
- long creationDate = dates[dates.length - 1 - i];
- Instant timestamp = Instant.ofEpochMilli(creationDate);
- Path expectedPath =
ContinuousJfrStarter.getRecordingPath(tempDirectory, "test", timestamp);
- Assertions.assertThat(remainingFiles)
- .describedAs("Should contain the expected file: %s",
expectedPath)
- .contains(expectedPath);
- }
+ private static boolean waitForRecordingPresence(String recordingName,
boolean expectedPresent, int maxAttempts,
+ long delayMs) {
+ for (int i = 0; i < maxAttempts; i++) {
+ boolean present = isRecordingPresent(recordingName);
+ if (present == expectedPresent) {
+ return true;
}
+ try {
+ Thread.sleep(delayMs);
Review Comment:
nit: can use `TestUtils::waitForCondition` pattern instead?
--
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.
To unsubscribe, e-mail: [email protected]
For queries about this service, please contact Infrastructure at:
[email protected]
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]