Skip to content

Commit

Permalink
[stats] Do not expose meaningless stats for publisher (#11454)
Browse files Browse the repository at this point in the history
### Motivation

Currently, publisher stats includes some fields that are meaningless for users.
```
count
producerNameOffset
producerNameLength
addressOffset
addressLength
connectedSinceOffset
connectedSinceLength
clientVersionOffset
clientVersionLength
stringBuffer
```
These cause the size of the json data to grow and should not be exposed. 

### Modifications

Add the `@JsonIgnore` annotation to the above fields. This is a modification similar to #11005.
  • Loading branch information
Masahiro Sakamoto committed Jul 28, 2021
1 parent 00ad07d commit d3c44ba
Show file tree
Hide file tree
Showing 5 changed files with 53 additions and 23 deletions.
Expand Up @@ -38,6 +38,7 @@
import org.testng.annotations.BeforeMethod;
import org.testng.annotations.Test;

import java.util.Iterator;
import java.util.List;
import java.util.Set;
import java.util.concurrent.TimeUnit;
Expand Down Expand Up @@ -193,7 +194,10 @@ public void testConsumerStatsOutput() throws Exception {
"lastAckedTimestamp",
"lastConsumedTimestamp",
"keyHashRanges",
"metadata");
"metadata",
"address",
"connectedSince",
"clientVersion");

final String topicName = "persistent://prop/use/ns-abc/testConsumerStatsOutput";
final String subName = "my-subscription";
Expand All @@ -208,9 +212,10 @@ public void testConsumerStatsOutput() throws Exception {
ObjectMapper mapper = ObjectMapperFactory.create();
JsonNode node = mapper.readTree(mapper.writer().writeValueAsString(stats.getSubscriptions()
.get(subName).getConsumers().get(0)));
if (node.fieldNames().hasNext()) {
String field = node.fieldNames().next();
Assert.assertTrue(allowedFields.contains(field));
Iterator<String> itr = node.fieldNames();
while (itr.hasNext()) {
String field = itr.next();
Assert.assertTrue(allowedFields.contains(field), field + " should not be exposed");
}

consumer.close();
Expand Down
Expand Up @@ -18,6 +18,7 @@
*/
package org.apache.pulsar.common.policies.data.stats;

import com.fasterxml.jackson.annotation.JsonIgnore;
import lombok.Data;
import org.apache.pulsar.client.api.ProducerAccessMode;
import org.apache.pulsar.common.policies.data.PublisherStats;
Expand All @@ -28,6 +29,7 @@
*/
@Data
public class PublisherStatsImpl implements PublisherStats {
@JsonIgnore
private int count;

public ProducerAccessMode accessMode;
Expand All @@ -48,25 +50,34 @@ public class PublisherStatsImpl implements PublisherStats {
public long producerId;

/** Producer name. */
@JsonIgnore
private int producerNameOffset = -1;
@JsonIgnore
private int producerNameLength;

/** Address of this publisher. */
@JsonIgnore
private int addressOffset = -1;
@JsonIgnore
private int addressLength;

/** Timestamp of connection. */
@JsonIgnore
private int connectedSinceOffset = -1;
@JsonIgnore
private int connectedSinceLength;

/** Client library version. */
@JsonIgnore
private int clientVersionOffset = -1;
@JsonIgnore
private int clientVersionLength;

/**
* In order to prevent multiple string objects under stats: create a string-buffer that stores data for all string
* place-holders.
*/
@JsonIgnore
private StringBuilder stringBuffer = new StringBuilder();

/** Metadata (key/value strings) associated with this publisher. */
Expand Down
Expand Up @@ -18,6 +18,7 @@
*/
package org.apache.pulsar.common.policies.data.stats;

import com.fasterxml.jackson.annotation.JsonIgnore;
import lombok.AccessLevel;
import lombok.Data;
import lombok.Getter;
Expand All @@ -37,6 +38,7 @@
*/
@Data
public class TopicStatsImpl implements TopicStats {
@JsonIgnore
private int count;

/** Total rate of messages published on the topic (msg/s). */
Expand Down
Expand Up @@ -20,14 +20,35 @@

import static org.testng.Assert.assertEquals;
import static org.testng.Assert.assertNull;
import static org.testng.Assert.assertTrue;

import com.fasterxml.jackson.databind.JsonNode;
import com.fasterxml.jackson.databind.ObjectMapper;
import com.google.common.collect.Sets;
import java.util.Iterator;
import java.util.Set;
import org.apache.pulsar.common.policies.data.stats.PublisherStatsImpl;
import org.apache.pulsar.common.util.ObjectMapperFactory;
import org.testng.annotations.Test;

public class PublisherStatsTest {

@Test
public void testPublisherStats() {
public void testPublisherStats() throws Exception {
Set<String> allowedFields = Sets.newHashSet(
"accessMode",
"msgRateIn",
"msgThroughputIn",
"averageMsgSize",
"chunkedMessageRate",
"producerId",
"metadata",
"address",
"connectedSince",
"clientVersion",
"producerName"
);

PublisherStatsImpl stats = new PublisherStatsImpl();
assertNull(stats.getAddress());
assertNull(stats.getClientVersion());
Expand All @@ -53,6 +74,15 @@ public void testPublisherStats() {
assertEquals(stats.getConnectedSince(), "connected");
assertEquals(stats.getAddress(), "address1");
assertEquals(stats.getClientVersion(), "version");

// Check if private fields are included in json
ObjectMapper mapper = ObjectMapperFactory.create();
JsonNode node = mapper.readTree(mapper.writer().writeValueAsString(stats));
Iterator<String> itr = node.fieldNames();
while (itr.hasNext()) {
String field = itr.next();
assertTrue(allowedFields.contains(field), field + " should not be exposed");
}

stats.setAddress(null);
assertNull(stats.getAddress());
Expand Down
18 changes: 0 additions & 18 deletions site2/docs/getting-started-docker.md
Expand Up @@ -106,7 +106,6 @@ The output is something like this:

```json
{
"count": 0,
"msgRateIn": 0.0,
"msgThroughputIn": 0.0,
"msgRateOut": 1.8332950480217471,
Expand All @@ -122,22 +121,12 @@ The output is something like this:
"offloadedStorageSize": 0,
"publishers": [
{
"count": 0,
"accessMode": "Shared",
"msgRateIn": 0.0,
"msgThroughputIn": 0.0,
"averageMsgSize": 0.0,
"chunkedMessageRate": 0.0,
"producerId": 0,
"producerNameOffset": 47,
"producerNameLength": 14,
"addressOffset": 0,
"addressLength": 16,
"connectedSinceOffset": 16,
"connectedSinceLength": 26,
"clientVersionOffset": 42,
"clientVersionLength": 5,
"stringBuffer": "/127.0.0.1:356042021-07-04T09:05:43.04788Z2.8.0standalone-2-5",
"metadata": {},
"address": "/127.0.0.1:35604",
"connectedSince": "2021-07-04T09:05:43.04788Z",
Expand Down Expand Up @@ -182,16 +171,9 @@ The output is something like this:
"unackedMessages": 0,
"avgMessagesPerEntry": 6,
"blockedConsumerOnUnackedMsgs": false,
"addressOffset": 0,
"addressLength": 16,
"connectedSinceOffset": 16,
"connectedSinceLength": 27,
"clientVersionOffset": 43,
"clientVersionLength": 5,
"lastAckedTimestamp": 1625389546162,
"lastConsumedTimestamp": 1625389546070,
"metadata": {},
"stringBuffer": "/127.0.0.1:354722021-07-04T08:58:21.287682Z2.8.0",
"address": "/127.0.0.1:35472",
"connectedSince": "2021-07-04T08:58:21.287682Z",
"clientVersion": "2.8.0"
Expand Down

0 comments on commit d3c44ba

Please sign in to comment.