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

penghui pushed a commit to branch branch-2.7
in repository https://gitbox.apache.org/repos/asf/pulsar.git

commit 257f60a7a020b7427da055c82f1bd40431096a9d
Author: Ming <[email protected]>
AuthorDate: Tue Dec 22 12:53:03 2020 -0500

    remove duplicated broker prometheus metrics type (#8995)
    
    ### Motivation
    
    If there are multiple topics from different namespaces, the broker 
prometheus metrics will print out duplicated `# TYPE` definition for 
pulsar_ml_AddEntryBytesRate and other managed ledger metrics.
    
    In fact, this problem can be verified by `promtool` 
https://github.com/prometheus/prometheus#building-from-source
    
    On the broker, run this command to check validity of Pulsar broker metric 
format.
    `curl localhost:8080/metrics/ | ~/go/bin/promtool check metrics`
    
    ### Modifications
    
    To prevent duplicated metrics type definition, the definition is now 
tracked and only printed out once. It leverages the existing metrics name Set 
already defined under parseMetricsToPrometheusMetrics() in 
PrometheusMetricsGenerator.java
    
    ### Verifying this change
    
    - [ x] Make sure that the change passes the CI checks.
    
    This change added tests and can be verified as follows:
    
    Added two topics under new namespaces to trigger conditions that duplicated 
prometheus type could happen previously under testManagedLedgerStats() of 
PrometheusMetricsTest.java. Updated test cases checks this duplicated type 
problem.
    
    (cherry picked from commit 73198195efc6f25e162840451e054473daf25f17)
---
 .../prometheus/PrometheusMetricsGenerator.java     | 12 +++--
 .../pulsar/broker/stats/PrometheusMetricsTest.java | 53 ++++++++++++++++++++--
 2 files changed, 58 insertions(+), 7 deletions(-)

diff --git 
a/pulsar-broker/src/main/java/org/apache/pulsar/broker/stats/prometheus/PrometheusMetricsGenerator.java
 
b/pulsar-broker/src/main/java/org/apache/pulsar/broker/stats/prometheus/PrometheusMetricsGenerator.java
index 39e1440..b7fae4b 100644
--- 
a/pulsar-broker/src/main/java/org/apache/pulsar/broker/stats/prometheus/PrometheusMetricsGenerator.java
+++ 
b/pulsar-broker/src/main/java/org/apache/pulsar/broker/stats/prometheus/PrometheusMetricsGenerator.java
@@ -150,9 +150,15 @@ public class PrometheusMetricsGenerator {
                         continue;
                     }
                 } else {
-                    stream.write("# TYPE 
").write(entry.getKey().replace("brk_", "pulsar_")).write(' ')
-                            .write(getTypeStr(metricType)).write('\n');
-                    stream.write(entry.getKey().replace("brk_", "pulsar_"))
+
+
+                    String name = entry.getKey();
+                    if (!names.contains(name)) {
+                        stream.write("# TYPE 
").write(entry.getKey().replace("brk_", "pulsar_")).write(' ')
+                                .write(getTypeStr(metricType)).write('\n');
+                        names.add(name);
+                    }
+                    stream.write(name.replace("brk_", "pulsar_"))
                             .write("{cluster=\"").write(cluster).write('"');
                 }
 
diff --git 
a/pulsar-broker/src/test/java/org/apache/pulsar/broker/stats/PrometheusMetricsTest.java
 
b/pulsar-broker/src/test/java/org/apache/pulsar/broker/stats/PrometheusMetricsTest.java
index 96e396f..b5e780e 100644
--- 
a/pulsar-broker/src/test/java/org/apache/pulsar/broker/stats/PrometheusMetricsTest.java
+++ 
b/pulsar-broker/src/test/java/org/apache/pulsar/broker/stats/PrometheusMetricsTest.java
@@ -471,10 +471,14 @@ public class PrometheusMetricsTest extends BrokerTestBase 
{
     public void testManagedLedgerStats() throws Exception {
         Producer<byte[]> p1 = 
pulsarClient.newProducer().topic("persistent://my-property/use/my-ns/my-topic1").create();
         Producer<byte[]> p2 = 
pulsarClient.newProducer().topic("persistent://my-property/use/my-ns/my-topic2").create();
+        Producer<byte[]> p3 = 
pulsarClient.newProducer().topic("persistent://my-property/use/my-ns2/my-topic1").create();
+        Producer<byte[]> p4 = 
pulsarClient.newProducer().topic("persistent://my-property/use/my-ns2/my-topic2").create();
         for (int i = 0; i < 10; i++) {
             String message = "my-message-" + i;
             p1.send(message.getBytes());
             p2.send(message.getBytes());
+            p3.send(message.getBytes());
+            p4.send(message.getBytes());
         }
 
         ByteArrayOutputStream statsOut = new ByteArrayOutputStream();
@@ -487,18 +491,59 @@ public class PrometheusMetricsTest extends BrokerTestBase 
{
                 System.out.println(e.getKey() + ": " + e.getValue())
         );
 
+        Map<String, String> typeDefs = new HashMap<String, String>();
+        Map<String, String> metricNames = new HashMap<String, String>();
+
+        Pattern typePattern = 
Pattern.compile("^#\\s+TYPE\\s+(\\w+)\\s+(\\w+)");
+        Pattern metricNamePattern = Pattern.compile("^(\\w+)\\{.+");
+
+        Splitter.on("\n").split(metricsStr).forEach(line -> {
+            if (line.isEmpty()) {
+                return;
+            }
+            if (line.startsWith("#")) {
+                // Check for duplicate type definitions
+                Matcher typeMatcher = typePattern.matcher(line);
+                checkArgument(typeMatcher.matches());
+                String metricName = typeMatcher.group(1);
+                String type = typeMatcher.group(2);
+
+                // From 
https://github.com/prometheus/docs/blob/master/content/docs/instrumenting/exposition_formats.md
+                // "Only one TYPE line may exist for a given metric name."
+                if (!typeDefs.containsKey(metricName)) {
+                    typeDefs.put(metricName, type);
+                } else {
+                    fail("Duplicate type definition found for TYPE definition 
" + metricName);
+                }
+                // From 
https://github.com/prometheus/docs/blob/master/content/docs/instrumenting/exposition_formats.md
+                // "The TYPE line for a metric name must appear before the 
first sample is reported for that metric name."
+                if (metricNames.containsKey(metricName)) {
+                    fail("TYPE definition for " + metricName + " appears after 
first sample");
+                }
+            } else {
+                Matcher metricMatcher = metricNamePattern.matcher(line);
+                checkArgument(metricMatcher.matches());
+                String metricName = metricMatcher.group(1);
+                metricNames.put(metricName, metricName);
+            }
+        });
+
         List<Metric> cm = (List<Metric>) 
metrics.get("pulsar_ml_AddEntryBytesRate");
-        assertEquals(cm.size(), 1);
+        assertEquals(cm.size(), 2);
         assertEquals(cm.get(0).tags.get("cluster"), "test");
-        assertEquals(cm.get(0).tags.get("namespace"), "my-property/use/my-ns");
+        String ns = cm.get(0).tags.get("namespace");
+        assertEquals(ns.equals("my-property/use/my-ns") || 
ns.equals("my-property/use/my-ns2"), true);
 
         cm = (List<Metric>) metrics.get("pulsar_ml_AddEntryMessagesRate");
-        assertEquals(cm.size(), 1);
+        assertEquals(cm.size(), 2);
         assertEquals(cm.get(0).tags.get("cluster"), "test");
-        assertEquals(cm.get(0).tags.get("namespace"), "my-property/use/my-ns");
+        ns = cm.get(0).tags.get("namespace");
+        assertEquals(ns.equals("my-property/use/my-ns") || 
ns.equals("my-property/use/my-ns2"), true);
 
         p1.close();
         p2.close();
+        p3.close();
+        p4.close();
     }
 
     @Test

Reply via email to