You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@nifi.apache.org by mc...@apache.org on 2019/10/14 15:33:32 UTC

[nifi] branch master updated: NIFI-6772: Improved the information that is emitted about Garbage Collection when a Diagnostics Dump is performed

This is an automated email from the ASF dual-hosted git repository.

mcgilman pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/nifi.git


The following commit(s) were added to refs/heads/master by this push:
     new 9dd0dda  NIFI-6772: Improved the information that is emitted about Garbage Collection when a Diagnostics Dump is performed
9dd0dda is described below

commit 9dd0dda6880ca473b230e5325920834abc213627
Author: Mark Payne <ma...@hotmail.com>
AuthorDate: Fri Oct 11 14:08:13 2019 -0400

    NIFI-6772: Improved the information that is emitted about Garbage Collection when a Diagnostics Dump is performed
    
    This closes #3809
---
 .../org/apache/nifi/controller/FlowController.java |  13 ++
 .../nifi/controller/GarbageCollectionEvent.java    |  44 +++++++
 .../nifi/controller/GarbageCollectionLog.java      |  32 +++++
 .../controller/RingBufferGarbageCollectionLog.java | 135 ++++++++++++++++++++
 .../controller/StandardGarbageCollectionEvent.java | 141 +++++++++++++++++++++
 .../bootstrap/BootstrapDiagnosticsFactory.java     |   3 +-
 .../tasks/GarbageCollectionDiagnosticTask.java     |  28 +++-
 7 files changed, 393 insertions(+), 3 deletions(-)

diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
index a5c1981..a6142ec 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
@@ -47,6 +47,7 @@ import java.util.concurrent.atomic.AtomicReference;
 import java.util.concurrent.locks.ReentrantReadWriteLock;
 import java.util.stream.Collectors;
 
+import javax.management.NotificationEmitter;
 import javax.net.ssl.SSLContext;
 
 import org.apache.commons.lang3.StringUtils;
@@ -307,6 +308,7 @@ public class FlowController implements ReportingTaskProvider, Authorizable, Node
     private final ParameterContextManager parameterContextManager;
     private final StandardFlowManager flowManager;
     private final RepositoryContextFactory repositoryContextFactory;
+    private final RingBufferGarbageCollectionLog gcLog;
 
     /**
      * true if controller is configured to operate in a clustered environment
@@ -470,6 +472,13 @@ public class FlowController implements ReportingTaskProvider, Authorizable, Node
         flowFileEventRepository = flowFileEventRepo;
         counterRepositoryRef = new AtomicReference<>(new StandardCounterRepository());
 
+        gcLog = new RingBufferGarbageCollectionLog(1000, 20L);
+        for (final GarbageCollectorMXBean mxBean : ManagementFactory.getGarbageCollectorMXBeans()) {
+            if (mxBean instanceof NotificationEmitter) {
+                ((NotificationEmitter) mxBean).addNotificationListener(gcLog, null, null);
+            }
+        }
+
         bulletinRepository = bulletinRepo;
         this.variableRegistry = variableRegistry == null ? VariableRegistry.EMPTY_REGISTRY : variableRegistry;
 
@@ -1790,6 +1799,10 @@ public class FlowController implements ReportingTaskProvider, Authorizable, Node
         return flowManager;
     }
 
+    public GarbageCollectionLog getGarbageCollectionLog() {
+        return gcLog;
+    }
+
     public RepositoryContextFactory getRepositoryContextFactory() {
         return repositoryContextFactory;
     }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/GarbageCollectionEvent.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/GarbageCollectionEvent.java
new file mode 100644
index 0000000..06f6bf8
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/GarbageCollectionEvent.java
@@ -0,0 +1,44 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.nifi.controller;
+
+import java.util.List;
+
+public interface GarbageCollectionEvent {
+    String getGarbageCollectorName();
+
+    String getAction();
+
+    String getCause();
+
+    long getStartTime();
+
+    long getEndTime();
+
+    long getDuration();
+
+    List<GarbageCollectionHeapSize> getHeapSizes();
+
+
+    interface GarbageCollectionHeapSize {
+        String getMemoryPoolName();
+
+        long getUsedBeforeCollection();
+
+        long getUsedAfterCollection();
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/GarbageCollectionLog.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/GarbageCollectionLog.java
new file mode 100644
index 0000000..0a8fa90
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/GarbageCollectionLog.java
@@ -0,0 +1,32 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.nifi.controller;
+
+import java.util.List;
+import java.util.Map;
+
+public interface GarbageCollectionLog {
+    long getMinDurationThreshold();
+
+    List<GarbageCollectionEvent> getGarbageCollectionEvents();
+
+    Map<String, Long> getGarbageCollectionCounts();
+
+    Map<String, Long> getAverageGarbageCollectionDurations();
+
+    GarbageCollectionEvent getLongestGarbageCollectionEvent();
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/RingBufferGarbageCollectionLog.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/RingBufferGarbageCollectionLog.java
new file mode 100644
index 0000000..817ed6a
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/RingBufferGarbageCollectionLog.java
@@ -0,0 +1,135 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.nifi.controller;
+
+import com.sun.management.GarbageCollectionNotificationInfo;
+import com.sun.management.GcInfo;
+import org.apache.nifi.util.RingBuffer;
+import org.apache.nifi.util.Tuple;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import javax.management.Notification;
+import javax.management.NotificationListener;
+import javax.management.openmbean.CompositeData;
+import java.lang.management.ManagementFactory;
+import java.lang.management.MemoryUsage;
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+public class RingBufferGarbageCollectionLog implements GarbageCollectionLog, NotificationListener {
+    private static final Logger logger = LoggerFactory.getLogger(RingBufferGarbageCollectionLog.class);
+    private final RingBuffer<GarbageCollectionEvent> events;
+    private final long minDurationThreshold;
+    private final long jvmStartTime;
+
+    // guarded by synchronizing on this
+    private GarbageCollectionEvent maxDurationEvent;
+    private final Map<String, Tuple<Long, Long>> timeAndCountPerAction = new HashMap<>();
+
+    public RingBufferGarbageCollectionLog(final int eventCount, final long minDurationThreshold) {
+        this.events = new RingBuffer<>(eventCount);
+        this.minDurationThreshold = minDurationThreshold;
+        jvmStartTime = ManagementFactory.getRuntimeMXBean().getStartTime();
+    }
+
+    @Override
+    public long getMinDurationThreshold() {
+        return minDurationThreshold;
+    }
+
+    @Override
+    public List<GarbageCollectionEvent> getGarbageCollectionEvents() {
+        return events.asList();
+    }
+
+    @Override
+    public synchronized Map<String, Long> getGarbageCollectionCounts() {
+        final Map<String, Long> counts = new HashMap<>();
+        timeAndCountPerAction.forEach((action, tuple) -> counts.put(action, tuple.getValue()));
+        return counts;
+    }
+
+    @Override
+    public synchronized Map<String, Long> getAverageGarbageCollectionDurations() {
+        final Map<String, Long> counts = new HashMap<>();
+        timeAndCountPerAction.forEach((action, tuple) -> counts.put(action, tuple.getKey() / tuple.getValue()));
+        return counts;
+    }
+
+    @Override
+    public synchronized GarbageCollectionEvent getLongestGarbageCollectionEvent() {
+        return maxDurationEvent;
+    }
+
+    @Override
+    public void handleNotification(final Notification notification, final Object handback) {
+        if (!notification.getType().equals(GarbageCollectionNotificationInfo.GARBAGE_COLLECTION_NOTIFICATION)) {
+            return;
+        }
+
+        final CompositeData compositeData = (CompositeData) notification.getUserData();
+        final GarbageCollectionNotificationInfo gcNotification = GarbageCollectionNotificationInfo.from(compositeData);
+        final GcInfo gcInfo = gcNotification.getGcInfo();
+
+        final String gcName = gcNotification.getGcName();
+        final String action = gcNotification.getGcAction();
+        final String cause = gcNotification.getGcCause();
+
+        final long startTime = jvmStartTime + gcInfo.getStartTime();
+        final long endTime = jvmStartTime + gcInfo.getEndTime();
+
+        final Map<String, MemoryUsage> usageAfter = gcInfo.getMemoryUsageAfterGc();
+        final Map<String, MemoryUsage> usageBefore = gcInfo.getMemoryUsageBeforeGc();
+
+        final List<GarbageCollectionEvent.GarbageCollectionHeapSize> heapSizes = new ArrayList<>();
+        for (final Map.Entry<String, MemoryUsage> entry : usageAfter.entrySet()) {
+            final MemoryUsage before = usageBefore.get(entry.getKey());
+            if (before == null) {
+                continue;
+            }
+
+            final MemoryUsage after = entry.getValue();
+            if (after.getUsed() == before.getUsed()) {
+                continue;
+            }
+
+            heapSizes.add(new StandardGarbageCollectionEvent.StandardGarbageCollectionHeapSize(entry.getKey(), before.getUsed(), after.getUsed()));
+        }
+
+        final GarbageCollectionEvent event = new StandardGarbageCollectionEvent(gcName, action, cause, startTime, endTime, heapSizes);
+
+        if (gcInfo.getDuration() >= minDurationThreshold) {
+            events.add(event);
+        }
+
+        synchronized (this) {
+            final Tuple<Long, Long> previousTuple = timeAndCountPerAction.get(action);
+            if (previousTuple == null){
+                timeAndCountPerAction.put(action, new Tuple<>(gcInfo.getDuration(), 1L));
+            } else {
+                timeAndCountPerAction.put(action, new Tuple<>(gcInfo.getDuration() + previousTuple.getKey(), 1L + previousTuple.getValue()));
+            }
+
+            if (maxDurationEvent == null || event.getDuration() > maxDurationEvent.getDuration()) {
+                maxDurationEvent = event;
+            }
+        }
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardGarbageCollectionEvent.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardGarbageCollectionEvent.java
new file mode 100644
index 0000000..d697d7d
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardGarbageCollectionEvent.java
@@ -0,0 +1,141 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.nifi.controller;
+
+import org.apache.nifi.util.FormatUtils;
+
+import java.text.DateFormat;
+import java.text.NumberFormat;
+import java.text.SimpleDateFormat;
+import java.util.Collections;
+import java.util.Date;
+import java.util.List;
+
+public class StandardGarbageCollectionEvent implements GarbageCollectionEvent {
+    private final String gcName;
+    private final String action;
+    private final String cause;
+    private final long startTime;
+    private final long endTime;
+    private final List<GarbageCollectionHeapSize> heapSizes;
+    private final DateFormat dateFormat = new SimpleDateFormat("MM/dd/yyyy HH:mm:ss.SSS");
+
+    public StandardGarbageCollectionEvent(final String gcName, final String action, final String cause, final long startTime, final long endTime, final List<GarbageCollectionHeapSize> heapSizes) {
+        this.gcName = gcName;
+        this.action = action;
+        this.cause = cause;
+        this.startTime = startTime;
+        this.endTime = endTime;
+        this.heapSizes = heapSizes;
+    }
+
+    @Override
+    public String getGarbageCollectorName() {
+        return gcName;
+    }
+
+    @Override
+    public String getAction() {
+        return action;
+    }
+
+    @Override
+    public String getCause() {
+        return cause;
+    }
+
+    @Override
+    public long getStartTime() {
+        return startTime;
+    }
+
+    @Override
+    public long getEndTime() {
+        return endTime;
+    }
+
+    @Override
+    public long getDuration() {
+        return endTime - startTime;
+    }
+
+    @Override
+    public List<GarbageCollectionHeapSize> getHeapSizes() {
+        return Collections.unmodifiableList(heapSizes);
+    }
+
+    @Override
+    public String toString() {
+        final StringBuilder sb = new StringBuilder();
+        sb.append("GarbageCollectionEvent[collectorName=").append(gcName)
+            .append(", action=").append(action)
+            .append(", cause=").append(cause)
+            .append(", startTime=").append(dateFormat.format(new Date(startTime)))
+            .append(", endTime=").append(dateFormat.format(new Date(endTime)))
+            .append(", duration=").append(NumberFormat.getInstance().format(endTime - startTime))
+            .append(" ms, heap sizes={");
+
+        for (int i=0; i < heapSizes.size(); i++) {
+            final GarbageCollectionHeapSize size = heapSizes.get(i);
+
+            sb.append(size.getMemoryPoolName())
+                .append(": ")
+                .append(FormatUtils.formatDataSize(size.getUsedBeforeCollection()))
+                .append(" => ")
+                .append(FormatUtils.formatDataSize(size.getUsedAfterCollection()));
+
+            if (i < heapSizes.size() - 1) {
+                sb.append(", ");
+            }
+        }
+
+        sb.append("}]");
+        return sb.toString();
+    }
+
+    public static class StandardGarbageCollectionHeapSize implements GarbageCollectionHeapSize {
+        private final String memoryPoolName;
+        private final long usedBefore;
+        private final long usedAfter;
+
+        public StandardGarbageCollectionHeapSize(final String memoryPoolName, final long usedBefore, final long usedAfter) {
+            this.memoryPoolName = memoryPoolName;
+            this.usedBefore = usedBefore;
+            this.usedAfter = usedAfter;
+        }
+
+        @Override
+        public String getMemoryPoolName() {
+            return memoryPoolName;
+        }
+
+        @Override
+        public long getUsedBeforeCollection() {
+            return usedBefore;
+        }
+
+        @Override
+        public long getUsedAfterCollection() {
+            return usedAfter;
+        }
+
+        @Override
+        public String toString() {
+            return "HeapSize[memoryPool=" + memoryPoolName + ", " + FormatUtils.formatDataSize(usedBefore) + " => " + FormatUtils.formatDataSize(usedAfter) + "]";
+        }
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/BootstrapDiagnosticsFactory.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/BootstrapDiagnosticsFactory.java
index 65b18a1..27acb66 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/BootstrapDiagnosticsFactory.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/BootstrapDiagnosticsFactory.java
@@ -51,7 +51,6 @@ public class BootstrapDiagnosticsFactory implements DiagnosticsFactory {
 
     @Override
     public DiagnosticsDump create(final boolean verbose) {
-        // TODO: Allow for a 'verbose' flag to indicate scanning content repo
         final List<DiagnosticsDumpElement> dumpElements = new ArrayList<>();
         for (final DiagnosticTask dumpTask : getDiagnosticTasks()) {
             try {
@@ -76,7 +75,7 @@ public class BootstrapDiagnosticsFactory implements DiagnosticsFactory {
         tasks.add(new FlowConfigurationDiagnosticTask(flowController));
         tasks.add(new LongRunningProcessorTask(flowController));
         tasks.add(new ClusterDiagnosticTask(flowController));
-        tasks.add(new GarbageCollectionDiagnosticTask());
+        tasks.add(new GarbageCollectionDiagnosticTask(flowController));
         tasks.add(new MemoryPoolPeakUsageTask());
         tasks.add(new RepositoryDiagnosticTask(flowController));
         tasks.add(new ComponentCountTask(flowController));
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/tasks/GarbageCollectionDiagnosticTask.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/tasks/GarbageCollectionDiagnosticTask.java
index 8b8aa81..e805e6a 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/tasks/GarbageCollectionDiagnosticTask.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/diagnostics/bootstrap/tasks/GarbageCollectionDiagnosticTask.java
@@ -16,16 +16,26 @@
  */
 package org.apache.nifi.diagnostics.bootstrap.tasks;
 
-import org.apache.nifi.diagnostics.DiagnosticsDumpElement;
+import org.apache.nifi.controller.FlowController;
+import org.apache.nifi.controller.GarbageCollectionEvent;
+import org.apache.nifi.controller.GarbageCollectionLog;
 import org.apache.nifi.diagnostics.DiagnosticTask;
+import org.apache.nifi.diagnostics.DiagnosticsDumpElement;
 import org.apache.nifi.diagnostics.StandardDiagnosticsDumpElement;
 
 import java.lang.management.GarbageCollectorMXBean;
 import java.lang.management.ManagementFactory;
 import java.util.ArrayList;
+import java.util.Collections;
 import java.util.List;
 
 public class GarbageCollectionDiagnosticTask implements DiagnosticTask {
+    private final FlowController flowController;
+
+    public GarbageCollectionDiagnosticTask(final FlowController flowController) {
+        this.flowController = flowController;
+    }
+
     @Override
     public DiagnosticsDumpElement captureDump(final boolean verbose) {
         final List<GarbageCollectorMXBean> garbageCollectors = ManagementFactory.getGarbageCollectorMXBeans();
@@ -36,6 +46,22 @@ public class GarbageCollectionDiagnosticTask implements DiagnosticTask {
             details.add(garbageCollector.getName() + " Collection Time (ms) : " + garbageCollector.getCollectionTime());
         }
 
+        final GarbageCollectionLog gcLog = flowController.getGarbageCollectionLog();
+        final List<GarbageCollectionEvent> events = gcLog.getGarbageCollectionEvents();
+        details.add("");
+
+        details.add("Longest Garbage Collection Event: " + gcLog.getLongestGarbageCollectionEvent());
+        details.add("Number of Garbage Collection Events Per Action: " + gcLog.getGarbageCollectionCounts());
+        details.add("Average Duration of Garbage Collection Event Per Action: " + gcLog.getAverageGarbageCollectionDurations());
+
+        details.add("");
+        details.add("Last " + events.size() + " Garbage Collection Events that had a duration exceeding " + gcLog.getMinDurationThreshold() + " millis:");
+
+        Collections.reverse(events); // Reverse events so that we show the most recent events first
+        for (final GarbageCollectionEvent event : events) {
+            details.add(event.toString());
+        }
+
         return new StandardDiagnosticsDumpElement("Garbage Collection", details);
     }
 }