Skip to content

Commit bbdf04c

Browse files
authored
Fix: Remove expensive changelog topics (#37)
* feat: configure stores for lower starting times and less duplicated data * feat: configure stores for lower starting times and less duplicated data * chore: enabled easier tagging by vars * feat: configure stores for lower starting times and less duplicated data * fix: start stream before sending messages * chore: use always master, pass other as args
1 parent f1a804c commit bbdf04c

File tree

5 files changed

+115
-147
lines changed

5 files changed

+115
-147
lines changed

Dockerfile

+1-1
Original file line numberDiff line numberDiff line change
@@ -14,8 +14,8 @@
1414

1515
FROM alpine
1616

17+
ARG VERSION=master-SNAPSHOT
1718
ENV USER jeqo
18-
ENV VERSION 0.5.3
1919

2020
WORKDIR /zipkin
2121

Makefile

+4-4
Original file line numberDiff line numberDiff line change
@@ -3,7 +3,7 @@ all: build
33

44
OPEN := 'xdg-open'
55
MAVEN := './mvnw'
6-
VERSION := '0.5.4-SNAPSHOT'
6+
VERSION := 'master-SNAPSHOT'
77
IMAGE_NAME := 'jeqo/zipkin-kafka'
88

99
.PHONY: run
@@ -23,11 +23,11 @@ kafka-topics:
2323

2424
.PHONY: docker-build
2525
docker-build:
26-
docker build -t ${IMAGE_NAME}:latest .
27-
docker build -t ${IMAGE_NAME}:${VERSION} .
26+
docker build --build-arg VERSION=${VERSION} -t ${IMAGE_NAME}:latest .
27+
docker build --build-arg VERSION=${VERSION} -t ${IMAGE_NAME}:${VERSION} .
2828

2929
.PHONY: docker-push
30-
docker-push: docker-build
30+
docker-push:
3131
docker push ${IMAGE_NAME}:latest
3232
docker push ${IMAGE_NAME}:${VERSION}
3333

storage/src/main/java/zipkin2/storage/kafka/streams/DependencyStoreTopologySupplier.java

+11-9
Original file line numberDiff line numberDiff line change
@@ -55,15 +55,17 @@ public DependencyStoreTopologySupplier(String dependencyTopicName,
5555
StreamsBuilder builder = new StreamsBuilder();
5656

5757
// Dependency links window store
58-
builder.addStateStore(Stores.windowStoreBuilder(
59-
Stores.persistentWindowStore(
60-
DEPENDENCIES_STORE_NAME,
61-
dependencyTtl,
62-
dependencyWindowSize,
63-
false),
64-
Serdes.String(),
65-
dependencyLinkSerde
66-
));
58+
builder.addStateStore(
59+
// Disabling logging to avoid long starting times
60+
Stores.windowStoreBuilder(
61+
Stores.persistentWindowStore(
62+
DEPENDENCIES_STORE_NAME,
63+
dependencyTtl,
64+
dependencyWindowSize,
65+
false),
66+
Serdes.String(),
67+
dependencyLinkSerde
68+
).withLoggingDisabled());
6769
// Consume dependency links stream
6870
builder.stream(dependencyTopicName, Consumed.with(Serdes.String(), dependencyLinkSerde))
6971
// Storage

storage/src/main/java/zipkin2/storage/kafka/streams/TraceStoreTopologySupplier.java

+72-68
Original file line numberDiff line numberDiff line change
@@ -22,7 +22,6 @@
2222
import java.util.Set;
2323
import java.util.function.Supplier;
2424
import org.apache.kafka.common.serialization.Serdes;
25-
import org.apache.kafka.streams.KeyValue;
2625
import org.apache.kafka.streams.StreamsBuilder;
2726
import org.apache.kafka.streams.Topology;
2827
import org.apache.kafka.streams.kstream.Consumed;
@@ -79,99 +78,104 @@ public TraceStoreTopologySupplier(String spansTopicName, List<String> autoComple
7978
@Override public Topology get() {
8079
StreamsBuilder builder = new StreamsBuilder();
8180
builder
81+
// Logging disabled to avoid long starting times
8282
.addStateStore(Stores.keyValueStoreBuilder(
8383
Stores.persistentKeyValueStore(TRACES_STORE_NAME),
8484
Serdes.String(),
85-
spansSerde))
85+
spansSerde).withLoggingDisabled())
86+
// Disabling logging to avoid long starting times
8687
.addStateStore(Stores.keyValueStoreBuilder(
8788
Stores.persistentKeyValueStore(SPAN_IDS_BY_TS_STORE_NAME),
8889
Serdes.Long(),
89-
spanIdsSerde))
90+
spanIdsSerde).withLoggingDisabled())
91+
// In-memory as service names are bounded
9092
.addStateStore(Stores.keyValueStoreBuilder(
91-
Stores.persistentKeyValueStore(SERVICE_NAMES_STORE_NAME),
93+
Stores.inMemoryKeyValueStore(SERVICE_NAMES_STORE_NAME),
9294
Serdes.String(),
9395
Serdes.String()))
96+
// In-memory as span names are bounded
9497
.addStateStore(Stores.keyValueStoreBuilder(
95-
Stores.persistentKeyValueStore(SPAN_NAMES_STORE_NAME),
98+
Stores.inMemoryKeyValueStore(SPAN_NAMES_STORE_NAME),
9699
Serdes.String(),
97100
namesSerde))
101+
// In-memory as remote-service names are bounded
98102
.addStateStore(Stores.keyValueStoreBuilder(
99-
Stores.persistentKeyValueStore(REMOTE_SERVICE_NAMES_STORE_NAME),
103+
Stores.inMemoryKeyValueStore(REMOTE_SERVICE_NAMES_STORE_NAME),
100104
Serdes.String(),
101105
namesSerde))
106+
// Persistent as values could be unbounded
102107
.addStateStore(Stores.keyValueStoreBuilder(
103108
Stores.persistentKeyValueStore(AUTOCOMPLETE_TAGS_STORE_NAME),
104109
Serdes.String(),
105-
namesSerde));
110+
namesSerde).withLoggingDisabled());
106111
// Traces stream
107112
KStream<String, List<Span>> spansStream = builder
108-
.stream(spansTopicName, Consumed.with(Serdes.String(), spansSerde));
113+
.stream(
114+
spansTopicName,
115+
Consumed.with(Serdes.String(), spansSerde)
116+
.withOffsetResetPolicy(Topology.AutoOffsetReset.LATEST));
109117
// Store traces
110118
spansStream.process(() -> new Processor<String, List<Span>>() {
111-
ProcessorContext context;
112-
// Actual traces store
113-
KeyValueStore<String, List<Span>> tracesStore;
114-
// timestamp index for trace IDs
115-
KeyValueStore<Long, Set<String>> spanIdsByTsStore;
119+
ProcessorContext context;
120+
// Actual traces store
121+
KeyValueStore<String, List<Span>> tracesStore;
122+
// timestamp index for trace IDs
123+
KeyValueStore<Long, Set<String>> spanIdsByTsStore;
116124

117-
@Override public void init(ProcessorContext context) {
118-
this.context = context;
119-
tracesStore =
120-
(KeyValueStore<String, List<Span>>) context.getStateStore(TRACES_STORE_NAME);
121-
spanIdsByTsStore =
122-
(KeyValueStore<Long, Set<String>>) context.getStateStore(SPAN_IDS_BY_TS_STORE_NAME);
123-
// Retention scheduling
124-
context.schedule(
125-
traceTtlCheckInterval,
126-
PunctuationType.STREAM_TIME,
127-
timestamp -> {
128-
if (traceTtl.toMillis() > 0 &&
129-
tracesStore.approximateNumEntries() > minTracesStored) {
130-
// preparing range filtering
131-
long from = 0L;
132-
long to = timestamp - traceTtl.toMillis();
133-
long toMicro = to * 1000;
134-
// query traceIds active during period
135-
try (final KeyValueIterator<Long, Set<String>> all =
136-
spanIdsByTsStore.range(from, toMicro)) {
137-
int deletions = 0; // logging purpose
138-
while (all.hasNext()) {
139-
final KeyValue<Long, Set<String>> record = all.next();
140-
spanIdsByTsStore.delete(record.key); // clean timestamp index
141-
for (String traceId : record.value) {
142-
tracesStore.delete(traceId); // clean traces store
143-
deletions++;
144-
}
145-
}
146-
if (deletions > 0) {
147-
LOG.info("Traces deletion emitted: {}, older than {}",
148-
deletions,
149-
Instant.ofEpochMilli(to).atZone(ZoneId.systemDefault()));
150-
}
125+
@Override public void init(ProcessorContext context) {
126+
this.context = context;
127+
tracesStore =
128+
(KeyValueStore<String, List<Span>>) context.getStateStore(TRACES_STORE_NAME);
129+
spanIdsByTsStore =
130+
(KeyValueStore<Long, Set<String>>) context.getStateStore(SPAN_IDS_BY_TS_STORE_NAME);
131+
// Retention scheduling
132+
context.schedule(
133+
traceTtlCheckInterval,
134+
PunctuationType.STREAM_TIME,
135+
timestamp -> {
136+
if (traceTtl.toMillis() > 0 &&
137+
tracesStore.approximateNumEntries() > minTracesStored) {
138+
// preparing range filtering
139+
long from = 0L;
140+
long to = timestamp - traceTtl.toMillis();
141+
long toMicro = to * 1000;
142+
// query traceIds active during period
143+
try (final KeyValueIterator<Long, Set<String>> range =
144+
spanIdsByTsStore.range(from, toMicro)) {
145+
range.forEachRemaining(record -> {
146+
spanIdsByTsStore.delete(record.key); // clean timestamp index
147+
for (String traceId : record.value) {
148+
tracesStore.delete(traceId); // clean traces store
151149
}
152-
}
153-
});
154-
}
150+
});
151+
LOG.info("Traces deletion emitted at {}, approx. number of traces stored {} - partition: {}",
152+
Instant.ofEpochMilli(to).atZone(ZoneId.systemDefault()),
153+
tracesStore.approximateNumEntries(),
154+
context.partition());
155+
}
156+
}
157+
});
158+
}
155159

156-
@Override public void process(String traceId, List<Span> spans) {
157-
if (!spans.isEmpty()) {
158-
// Persist traces
159-
List<Span> currentSpans = tracesStore.get(traceId);
160-
if (currentSpans == null) currentSpans = new ArrayList<>();
161-
currentSpans.addAll(spans);
162-
tracesStore.put(traceId, currentSpans);
163-
// Persist timestamp indexed span ids
164-
long timestamp = spans.get(0).timestamp();
165-
Set<String> currentSpanIds = spanIdsByTsStore.get(timestamp);
166-
if (currentSpanIds == null) currentSpanIds = new HashSet<>();
167-
currentSpanIds.add(traceId);
168-
spanIdsByTsStore.put(timestamp, currentSpanIds);
169-
}
170-
}
160+
@Override public void process(String traceId, List<Span> spans) {
161+
if (!spans.isEmpty()) {
162+
// Persist traces
163+
List<Span> currentSpans = tracesStore.get(traceId);
164+
if (currentSpans == null) currentSpans = new ArrayList<>();
165+
currentSpans.addAll(spans);
166+
tracesStore.put(traceId, currentSpans);
167+
// Persist timestamp indexed span ids
168+
long timestamp = spans.get(0).timestamp();
169+
Set<String> currentSpanIds = spanIdsByTsStore.get(timestamp);
170+
if (currentSpanIds == null) currentSpanIds = new HashSet<>();
171+
currentSpanIds.add(traceId);
172+
spanIdsByTsStore.put(timestamp, currentSpanIds);
173+
}
174+
}
171175

172-
@Override public void close() {
173-
}
174-
}, TRACES_STORE_NAME, SPAN_IDS_BY_TS_STORE_NAME);
176+
@Override public void close() {
177+
}
178+
}, TRACES_STORE_NAME, SPAN_IDS_BY_TS_STORE_NAME);
175179
// Store service, span and remote service names
176180
spansStream.process(() -> new Processor<String, List<Span>>() {
177181
KeyValueStore<String, String> serviceNameStore;

storage/src/test/java/zipkin2/storage/kafka/KafkaStorageIT.java

+27-65
Original file line numberDiff line numberDiff line change
@@ -28,6 +28,7 @@
2828
import org.apache.kafka.clients.producer.ProducerRecord;
2929
import org.apache.kafka.common.serialization.ByteArrayDeserializer;
3030
import org.apache.kafka.common.serialization.StringSerializer;
31+
import org.apache.kafka.streams.KafkaStreams;
3132
import org.apache.kafka.streams.errors.InvalidStateStoreException;
3233
import org.apache.kafka.streams.integration.utils.IntegrationTestUtils;
3334
import org.junit.jupiter.api.AfterEach;
@@ -163,6 +164,11 @@ class KafkaStorageIT {
163164
.timestamp(TODAY * 1000 + 10).duration(8)
164165
.build();
165166
List<Span> spans = Arrays.asList(parent, child);
167+
// When: and stores running
168+
SpanStore spanStore = storage.spanStore();
169+
ServiceAndSpanNames serviceAndSpanNames = storage.serviceAndSpanNames();
170+
await().atMost(10, TimeUnit.SECONDS)
171+
.until(() -> storage.traceStoreStream.state().equals(KafkaStreams.State.RUNNING));
166172
// When: been published
167173
tracesProducer.send(new ProducerRecord<>(storage.spansTopicName, parent.traceId(), spans));
168174
tracesProducer.send(new ProducerRecord<>(storage.spansTopicName, other.traceId(),
@@ -171,76 +177,32 @@ class KafkaStorageIT {
171177
// Then: stored
172178
IntegrationTestUtils.waitUntilMinRecordsReceived(
173179
testConsumerConfig, storage.spansTopicName, 1, 10000);
174-
// When: and stores running
175-
SpanStore spanStore = storage.spanStore();
176-
ServiceAndSpanNames serviceAndSpanNames = storage.serviceAndSpanNames();
177180
// Then: services names are searchable
178-
await().atMost(10, TimeUnit.SECONDS)
179-
.until(() -> {
180-
List<List<Span>> traces = new ArrayList<>();
181-
try {
182-
traces =
183-
spanStore.getTraces(QueryRequest.newBuilder()
184-
.endTs(TODAY + 1)
185-
.lookback(Duration.ofSeconds(30).toMillis())
186-
.serviceName("svc_a")
187-
.limit(10)
188-
.build())
189-
.execute();
190-
} catch (InvalidStateStoreException e) { // ignoring state issues
191-
System.err.println(e.getMessage());
192-
} catch (Exception e) {
193-
e.printStackTrace();
194-
}
195-
return traces.size() == 1
196-
&& traces.get(0).size() == 2; // Trace is found and has two spans
197-
}); // wait for services to be available
198-
List<List<Span>> traces = new ArrayList<>();
199-
try {
200-
traces =
201-
spanStore.getTraces(QueryRequest.newBuilder()
202-
.endTs(TODAY + 1)
203-
.lookback(Duration.ofMinutes(1).toMillis())
204-
.limit(1)
205-
.build())
206-
.execute();
207-
} catch (InvalidStateStoreException e) { // ignoring state issues
208-
System.err.println(e.getMessage());
209-
} catch (Exception e) {
210-
e.printStackTrace();
211-
}
181+
List<List<Span>> traces = spanStore.getTraces(QueryRequest.newBuilder()
182+
.endTs(TODAY + 1)
183+
.lookback(Duration.ofSeconds(30).toMillis())
184+
.serviceName("svc_a")
185+
.limit(10)
186+
.build())
187+
.execute();
212188
assertEquals(1, traces.size());
213-
assertEquals(1, traces.get(0).size()); // last trace is returned first
214-
215-
List<String> services = new ArrayList<>();
216-
try {
217-
services = serviceAndSpanNames.getServiceNames().execute();
218-
} catch (InvalidStateStoreException e) { // ignoring state issues
219-
System.err.println(e.getMessage());
220-
} catch (Exception e) {
221-
e.printStackTrace();
222-
}
189+
assertEquals(traces.get(0).size(), 2); // Trace is found and has two spans
190+
List<List<Span>> filteredTraces =
191+
spanStore.getTraces(QueryRequest.newBuilder()
192+
.endTs(TODAY + 1)
193+
.lookback(Duration.ofMinutes(1).toMillis())
194+
.limit(1)
195+
.build())
196+
.execute();
197+
assertEquals(1, filteredTraces.size());
198+
assertEquals(1, filteredTraces.get(0).size()); // last trace is returned first
199+
List<String> services = serviceAndSpanNames.getServiceNames().execute();
223200
assertEquals(3, services.size()); // There are two service names
224-
225-
List<String> spanNames = new ArrayList<>();
226-
try {
227-
spanNames = serviceAndSpanNames.getSpanNames("svc_a")
228-
.execute();
229-
} catch (InvalidStateStoreException e) { // ignoring state issues
230-
System.err.println(e.getMessage());
231-
} catch (Exception e) {
232-
e.printStackTrace();
233-
}
201+
List<String> spanNames = serviceAndSpanNames.getSpanNames("svc_a")
202+
.execute();
234203
assertEquals(1, spanNames.size());// Service names have one span name
235204

236-
List<String> remoteServices = new ArrayList<>();
237-
try {
238-
remoteServices = serviceAndSpanNames.getRemoteServiceNames("svc_a").execute();
239-
} catch (InvalidStateStoreException e) { // ignoring state issues
240-
System.err.println(e.getMessage());
241-
} catch (Exception e) {
242-
e.printStackTrace();
243-
}
205+
List<String> remoteServices = serviceAndSpanNames.getRemoteServiceNames("svc_a").execute();
244206
assertEquals(1, remoteServices.size());// And one remote service name
245207
}
246208

0 commit comments

Comments
 (0)