Michael Melsen created KAFKA-8153:
-------------------------------------

             Summary: Streaming application with state stores takes up to 1 
hour to restart
                 Key: KAFKA-8153
                 URL: https://issues.apache.org/jira/browse/KAFKA-8153
             Project: Kafka
          Issue Type: Bug
          Components: streams
    Affects Versions: 2.1.1
            Reporter: Michael Melsen


We are using spring cloud stream with Kafka streams 2.0.1 and utilizing the 
InteractiveQueryService to fetch data from the stores. There are 4 stores that 
persist data on disk after aggregating data. The code for the topology looks 
like this:
{code:java}
@Slf4j
@EnableBinding(SensorMeasurementBinding.class)
public class Consumer {

  public static final String RETENTION_MS = "retention.ms";
  public static final String CLEANUP_POLICY = "cleanup.policy";

  @Value("${windowstore.retention.ms}")
  private String retention;

/**
 * Process the data flowing in from a Kafka topic. Aggregate the data to:
 * - 2 minute
 * - 15 minutes
 * - one hour
 * - 12 hours
 *
 * @param stream
 */
@StreamListener(SensorMeasurementBinding.ERROR_SCORE_IN)
public void process(KStream<String, SensorMeasurement> stream) {

    Map<String, String> topicConfig = new HashMap<>();
    topicConfig.put(RETENTION_MS, retention);
    topicConfig.put(CLEANUP_POLICY, "delete");

    log.info("Changelog and local window store retention.ms: {} and 
cleanup.policy: {}",
            topicConfig.get(RETENTION_MS),
            topicConfig.get(CLEANUP_POLICY));

    createWindowStore(LocalStore.TWO_MINUTES_STORE, topicConfig, stream);
    createWindowStore(LocalStore.FIFTEEN_MINUTES_STORE, topicConfig, stream);
    createWindowStore(LocalStore.ONE_HOUR_STORE, topicConfig, stream);
    createWindowStore(LocalStore.TWELVE_HOURS_STORE, topicConfig, stream);
  }

  private void createWindowStore(
        LocalStore localStore,
        Map<String, String> topicConfig,
        KStream<String, SensorMeasurement> stream) {

    // Configure how the statestore should be materialized using the provide 
storeName
    Materialized<String, ErrorScore, WindowStore<Bytes, byte[]>> materialized = 
Materialized
            .as(localStore.getStoreName());

    // Set retention of changelog topic
    materialized.withLoggingEnabled(topicConfig);

    // Configure how windows looks like and how long data will be retained in 
local stores
    TimeWindows configuredTimeWindows = getConfiguredTimeWindows(
            localStore.getTimeUnit(), 
Long.parseLong(topicConfig.get(RETENTION_MS)));

    // Processing description:
    // The input data are 'samples' with key 
<installationId>:<assetId>:<modelInstanceId>:<algorithmName>
    // 1. With the map we add the Tag to the key and we extract the error score 
from the data
    // 2. With the groupByKey we group  the data on the new key
    // 3. With windowedBy we split up the data in time intervals depending on 
the provided LocalStore enum
    // 4. With reduce we determine the maximum value in the time window
    // 5. Materialized will make it stored in a table
    stream
            .map(getInstallationAssetModelAlgorithmTagKeyMapper())
            .groupByKey()
            .windowedBy(configuredTimeWindows)
            .reduce((aggValue, newValue) -> getMaxErrorScore(aggValue, 
newValue), materialized);
  }

  private TimeWindows getConfiguredTimeWindows(long windowSizeMs, long 
retentionMs) {
    TimeWindows timeWindows = TimeWindows.of(windowSizeMs);
    timeWindows.until(retentionMs);
    return timeWindows;
  }

  /**
   * Determine the max error score to keep by looking at the aggregated error 
signal and
   * freshly consumed error signal
   *
   * @param aggValue
   * @param newValue
   * @return
   */
  private ErrorScore getMaxErrorScore(ErrorScore aggValue, ErrorScore newValue) 
{
    if(aggValue.getErrorSignal() > newValue.getErrorSignal()) {
        return aggValue;
    }
    return newValue;
  }

  private KeyValueMapper<String, SensorMeasurement,
        KeyValue<? extends String, ? extends ErrorScore>> 
getInstallationAssetModelAlgorithmTagKeyMapper() {
    return (s, sensorMeasurement) -> new KeyValue<>(s + "::" + 
sensorMeasurement.getT(),
            new ErrorScore(sensorMeasurement.getTs(), sensorMeasurement.getE(), 
sensorMeasurement.getO()));
  }
}
{code}
So we are materializing aggregated data to four different stores after 
determining the max value within a specific window for a specific key. Please 
note that retention which is set to two months of data and the clean up policy 
delete. We don't compact data.

The size of the individual state stores on disk is between 14 to 20 gb of data.

We are making use of Interactive Queries: 
[https://docs.confluent.io/current/streams/developer-guide/interactive-queries.html#interactive-queries]

On our setup we have 4 instances of our streaming app to be used as one 
consumer group. So every instance will store a specific part of all data in its 
store.

This all seems to work nicely. Until we restart one or more instances and wait 
for it to become available again. (Restart time only is about 3 minutes max). I 
would expect that the restart of the app would not take that long but 
unfortunately it takes op to 1 hour. I guess that the issue is caused by the 
fact that Streams starts restoring state stores, by first deleting the local 
state stores. I would have expected that as we persist the state store data on 
persisted volumes outside of the container that runs on kubernetes, the app 
would receive the last offset from the broker and only has to continue from 
that point as the previously consumed data is already there in the state store. 

Restarting our app triggers a restore task:
{code:java}
StreamThread-2] Restoring task 4_3's state store twelve-hours-error-score from 
beginning of the changelog 
anomaly-timeline-twelve-hours-error-score-changelog-3.{code}
Streamsconfig:
{code:java}
2019-03-25 09:14:54,352 INFO main org.apache.kafka.common.config.AbstractConfig 
StreamsConfig values:
application.id = anomaly-timeline
application.server = localhost:5000
bootstrap.servers = [localhost:9095]
buffered.records.per.partition = 1000
cache.max.bytes.buffering = 10485760
client.id = 
anomaly-timeline-bd6b4a26-9053-4670-9637-ebf4bfe87a96-StreamThread-1-consumer
commit.interval.ms = 30000
connections.max.idle.ms = 540000
default.deserialization.exception.handler = class 
org.apache.kafka.streams.errors.LogAndFailExceptionHandler
default.key.serde = class 
org.apache.kafka.common.serialization.Serdes$ByteArraySerde
default.production.exception.handler = class 
org.apache.kafka.streams.errors.DefaultProductionExceptionHandler
default.timestamp.extractor = class 
org.apache.kafka.streams.processor.FailOnInvalidTimestamp
default.value.serde = class 
org.apache.kafka.common.serialization.Serdes$ByteArraySerde
metadata.max.age.ms = 300000
metric.reporters = []
metrics.num.samples = 2
metrics.recording.level = INFO
metrics.sample.window.ms = 30000
num.standby.replicas = 2
num.stream.threads = 1
partition.grouper = class 
org.apache.kafka.streams.processor.DefaultPartitionGrouper
poll.ms = 100
processing.guarantee = at_least_once
receive.buffer.bytes = 32768
reconnect.backoff.max.ms = 1000
reconnect.backoff.ms = 50
replication.factor = 1
request.timeout.ms = 40000
retries = 0
retry.backoff.ms = 100
rocksdb.config.setter = null
security.protocol = PLAINTEXT
send.buffer.bytes = 131072
state.cleanup.delay.ms = 600000
state.dir = /tmp/kafka-streams
topology.optimization = none
upgrade.from = null
windowstore.changelog.additional.retention.ms = 86400000
{code}
In addition to figure out what exactly is happening, I've implemented a 
StateRestoreListener:
{code:java}
public class LoggingStateRestoreListener implements StateRestoreListener {
    private static final Logger LOG = 
LoggerFactory.getLogger(LoggingStateRestoreListener.class);
    private final Map<TopicPartition, Long> totalToRestore = new 
ConcurrentHashMap<>();
    private final Map<TopicPartition, Long> restoredSoFar = new 
ConcurrentHashMap<>();

    @Override
    public void onRestoreStart(TopicPartition topicPartition, String store, 
long start, long end) {
        long toRestore = end - start;
        totalToRestore.put(topicPartition, toRestore);
        LOG.info("Starting restoration for {} on topic-partition {} total to 
restore {}",
                store, topicPartition, toRestore);
    }

    @Override
    public void onBatchRestored(
            TopicPartition topicPartition,
            String storeName,
            long batchEndOffset,
            long batchCompleted) {

        NumberFormat formatter = new DecimalFormat("#.##");
        long currentProgress = batchCompleted + 
restoredSoFar.getOrDefault(topicPartition, 0L);
        double percentComplete = (double) currentProgress / 
totalToRestore.get(topicPartition);
        LOG.info("Completed {} for {}% of total restoration for {} on {}", 
batchCompleted,
                formatter.format(percentComplete * 100.00), storeName, 
topicPartition);
        restoredSoFar.put(topicPartition, currentProgress);
    }

    @Override
    public void onRestoreEnd(TopicPartition topicPartition, String store, long 
totalRestored) {
        LOG.info("Restoration completed for {} on topic-partition {}", store, 
topicPartition);
        restoredSoFar.put(topicPartition, 0L);
    }
}
{code}
This is invoked for same partition over and over again when starting the app: 
2019-03-22 09:35:16,03 INFO 
atl-737e402a-c5bb-446e-8dad-b81e918d4047-StreamThread-1 c.w.a.a.L Starting 
restoration for fifteen-min on topic-partition atl-fifteen-min-changelog-2 
total to restore 119673932



I've also enabled Trace logging, not sure if it is of any use:
{code:java}
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StoreChangelogReader stream-thread 
[anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1] Restored 
from anomaly-timeline-fifteen-minutes-error-score-changelog-3 to 
fifteen-minutes-error-score with 1000 records, ending offset is 12114866, next 
starting position is 12114867
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AssignedTasks stream-thread 
[anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1] stream 
task changelog partitions that have completed restoring so far: 
[anomaly-timeline-two-minutes-error-score-changelog-3, 
anomaly-timeline-twelve-hours-error-score-changelog-0, 
anomaly-timeline-two-minutes-error-score-changelog-1, 
anomaly-timeline-one-hour-error-score-changelog-2, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0, 
anomaly-timeline-one-hour-error-score-changelog-0, 
anomaly-timeline-one-hour-error-score-changelog-1]
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AssignedTasks stream-thread 
[anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1] stream 
task 1_2 cannot resume processing yet since some of its changelog partitions 
have not completed restoring: 
[anomaly-timeline-two-minutes-error-score-changelog-2]
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AssignedTasks stream-thread 
[anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1] stream 
task 2_3 cannot resume processing yet since some of its changelog partitions 
have not completed restoring: 
[anomaly-timeline-fifteen-minutes-error-score-changelog-3]
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AssignedTasks stream-thread 
[anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1] stream 
task 2_7 cannot resume processing yet since some of its changelog partitions 
have not completed restoring: 
[anomaly-timeline-fifteen-minutes-error-score-changelog-7]
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamThread stream-thread 
[anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1] 
Committing all active tasks [1_1, 2_0, 3_0, 2_1, 1_2, 1_3, 4_0, 3_1, 3_2, 2_3, 
2_7] and standby tasks [2_2, 4_1, 1_4, 4_2, 3_3, 2_4, 1_5, 3_4, 1_6, 4_4, 2_6, 
4_5, 3_7, 3_8, 4_8] since 504ms has elapsed (commit interval is 500ms)
2019-03-11 15:02:26,476 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [1_1] Committing
2019-03-11 15:02:26,476 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [1_1] Flushing 
state and producer
2019-03-11 15:02:26,476 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [1_1] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,477 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [1_1] 
Flushing store two-minutes-error-score
2019-03-11 15:02:26,477 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [1_1] 
Flushing producer
2019-03-11 15:02:26,477 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [1_1] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-scor
e-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,479 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [2_0] Committing
2019-03-11 15:02:26,479 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [2_0] Flushing 
state and producer
2019-03-11 15:02:26,479 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [2_0] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,479 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [2_0] 
Flushing store fifteen-minutes-error-score
2019-03-11 15:02:26,480 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [2_0] 
Flushing producer
2019-03-11 15:02:26,480 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [2_0] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-scor
e-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,482 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [3_0] Committing
2019-03-11 15:02:26,482 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [3_0] Flushing 
state and producer
2019-03-11 15:02:26,482 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_0] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,482 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_0] 
Flushing store one-hour-error-score
2019-03-11 15:02:26,482 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [3_0] 
Flushing producer
2019-03-11 15:02:26,482 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_0] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=111734366, 
anomaly-timeline-twelve-hours-error-score-changelog-2=18521526, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113136853, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-two-minutes-error-score-changelog-7=113207826, 
anomaly-timeline-two-minutes-error-score-changelog-5=14457271, 
anomaly-timeline-twelve-hours-error-score-changelog-7=5179040, 
anomaly-timeline-twelve-hours-error-score-changelog-5=115476239, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=25705510, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=25663789, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=113452888, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-s
core-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-7=112077749, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=109499390, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=112561985}
2019-03-11 15:02:26,484 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [2_1] Committing
2019-03-11 15:02:26,484 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [2_1] Flushing 
state and producer
2019-03-11 15:02:26,484 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [2_1] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,484 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [2_1] 
Flushing store fifteen-minutes-error-score
2019-03-11 15:02:26,484 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [2_1] 
Flushing producer
2019-03-11 15:02:26,485 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [2_1] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-scor
e-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,487 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [1_3] Committing
2019-03-11 15:02:26,487 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [1_3] Flushing 
state and producer
2019-03-11 15:02:26,487 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [1_3] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,487 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [1_3] 
Flushing store two-minutes-error-score
2019-03-11 15:02:26,487 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [1_3] 
Flushing producer
2019-03-11 15:02:26,488 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [1_3] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113136853, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113207826, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,490 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [4_0] Committing
2019-03-11 15:02:26,490 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [4_0] Flushing 
state and producer
2019-03-11 15:02:26,490 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [4_0] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,490 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [4_0] 
Flushing store twelve-hours-error-score
2019-03-11 15:02:26,490 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [4_0] 
Flushing producer
2019-03-11 15:02:26,490 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [4_0] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-scor
e-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,492 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [3_1] Committing
2019-03-11 15:02:26,492 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [3_1] Flushing 
state and producer
2019-03-11 15:02:26,492 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_1] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,492 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_1] 
Flushing store one-hour-error-score
2019-03-11 15:02:26,492 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [3_1] 
Flushing producer
2019-03-11 15:02:26,493 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_1] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-scor
e-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,495 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [3_2] Committing
2019-03-11 15:02:26,495 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StreamTask task [3_2] Flushing 
state and producer
2019-03-11 15:02:26,495 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_2] 
Flushing all stores registered in the state manager
2019-03-11 15:02:26,495 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_2] 
Flushing store one-hour-error-score
2019-03-11 15:02:26,495 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.RecordCollectorImpl task [3_2] 
Flushing producer
2019-03-11 15:02:26,495 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager task [3_2] 
Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113136853, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113207826, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,497 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [2_2] 
Committing
2019-03-11 15:02:26,497 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_2] Flushing all stores registered in the state manager
2019-03-11 15:02:26,497 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_2] Flushing store fifteen-minutes-error-score
2019-03-11 15:02:26,497 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_2] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=25705510, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=25663789, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=62148419, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,500 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_2] Updating store offset limit for partition 
anomaly-timeline-fifteen-minutes-error-score-repartition-2 to 125763438
2019-03-11 15:02:26,500 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [2_2] 
Updating store offset limits 125763438 for changelog 
anomaly-timeline-fifteen-minutes-error-score-repartition-2
2019-03-11 15:02:26,500 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [4_1] 
Committing
2019-03-11 15:02:26,500 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_1] Flushing all stores registered in the state manager
2019-03-11 15:02:26,500 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_1] Flushing store twelve-hours-error-score
2019-03-11 15:02:26,500 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_1] Writing checkpoint: 
{anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-er
ror-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,502 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_1] Updating store offset limit for partition 
anomaly-timeline-twelve-hours-error-score-repartition-1 to 124006951
2019-03-11 15:02:26,502 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [4_1] 
Updating store offset limits 124006951 for changelog 
anomaly-timeline-twelve-hours-error-score-repartition-1
2019-03-11 15:02:26,502 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [1_4] 
Committing
2019-03-11 15:02:26,503 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_4] Flushing all stores registered in the state manager
2019-03-11 15:02:26,503 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_4] Flushing store two-minutes-error-score
2019-03-11 15:02:26,503 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_4] Writing checkpoint: 
{anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-er
ror-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,505 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_4] Updating store offset limit for partition 
anomaly-timeline-two-minutes-error-score-repartition-4 to 126430132
2019-03-11 15:02:26,505 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [1_4] 
Updating store offset limits 126430132 for changelog 
anomaly-timeline-two-minutes-error-score-repartition-4
2019-03-11 15:02:26,506 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [4_2] 
Committing
2019-03-11 15:02:26,506 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_2] Flushing all stores registered in the state manager
2019-03-11 15:02:26,506 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_2] Flushing store twelve-hours-error-score
2019-03-11 15:02:26,506 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_2] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-2=9574079, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,508 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_2] Updating store offset limit for partition 
anomaly-timeline-twelve-hours-error-score-repartition-2 to 125763449
2019-03-11 15:02:26,508 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [4_2] 
Updating store offset limits 125763449 for changelog 
anomaly-timeline-twelve-hours-error-score-repartition-2
2019-03-11 15:02:26,508 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [3_3] 
Committing
2019-03-11 15:02:26,508 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_3] Flushing all stores registered in the state manager
2019-03-11 15:02:26,508 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_3] Flushing store one-hour-error-score
2019-03-11 15:02:26,508 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_3] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=111734366, 
anomaly-timeline-twelve-hours-error-score-changelog-2=34315444, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113136853, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113207826, 
anomaly-timeline-twelve-hours-error-score-changelog-7=5179040, 
anomaly-timeline-twelve-hours-error-score-changelog-5=115476239, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=123200563, 
anomaly-timeline-one-hour-error-score-changelog-3=51371192, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=113452888, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=109499390, 
anomaly-timeline-
fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=112561985}
2019-03-11 15:02:26,511 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_3] Updating store offset limit for partition 
anomaly-timeline-one-hour-error-score-repartition-3 to 118142814
2019-03-11 15:02:26,511 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [3_3] 
Updating store offset limits 118142814 for changelog 
anomaly-timeline-one-hour-error-score-repartition-3
2019-03-11 15:02:26,511 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [2_4] 
Committing
2019-03-11 15:02:26,511 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_4] Flushing all stores registered in the state manager
2019-03-11 15:02:26,511 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_4] Flushing store fifteen-minutes-error-score
2019-03-11 15:02:26,511 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_4] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,513 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_4] Updating store offset limit for partition 
anomaly-timeline-fifteen-minutes-error-score-repartition-4 to 126430126
2019-03-11 15:02:26,513 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [2_4] 
Updating store offset limits 126430126 for changelog 
anomaly-timeline-fifteen-minutes-error-score-repartition-4
2019-03-11 15:02:26,513 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [1_5] 
Committing
2019-03-11 15:02:26,514 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_5] Flushing all stores registered in the state manager
2019-03-11 15:02:26,514 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_5] Flushing store two-minutes-error-score
2019-03-11 15:02:26,514 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_5] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-5=9284252, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,516 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_5] Updating store offset limit for partition 
anomaly-timeline-two-minutes-error-score-repartition-5 to 118596515
2019-03-11 15:02:26,516 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [1_5] 
Updating store offset limits 118596515 for changelog 
anomaly-timeline-two-minutes-error-score-repartition-5
2019-03-11 15:02:26,516 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [3_4] 
Committing
2019-03-11 15:02:26,516 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_4] Flushing all stores registered in the state manager
2019-03-11 15:02:26,517 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_4] Flushing store one-hour-error-score
2019-03-11 15:02:26,517 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_4] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9064711, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,519 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_4] Updating store offset limit for partition 
anomaly-timeline-one-hour-error-score-repartition-4 to 126430139
2019-03-11 15:02:26,519 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [3_4] 
Updating store offset limits 126430139 for changelog 
anomaly-timeline-one-hour-error-score-repartition-4
2019-03-11 15:02:26,519 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [1_6] 
Committing
2019-03-11 15:02:26,519 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_6] Flushing all stores registered in the state manager
2019-03-11 15:02:26,519 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_6] Flushing store two-minutes-error-score
2019-03-11 15:02:26,519 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_6] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124474512, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,522 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[1_6] Updating store offset limit for partition 
anomaly-timeline-two-minutes-error-score-repartition-6 to 115826992
2019-03-11 15:02:26,522 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [1_6] 
Updating store offset limits 115826992 for changelog 
anomaly-timeline-two-minutes-error-score-repartition-6
2019-03-11 15:02:26,522 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [4_4] 
Committing
2019-03-11 15:02:26,522 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_4] Flushing all stores registered in the state manager
2019-03-11 15:02:26,522 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_4] Flushing store twelve-hours-error-score
2019-03-11 15:02:26,522 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_4] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-two-minutes-error-score-changelog-0=112678210, 
anomaly-timeline-twelve-hours-error-score-changelog-2=4745702, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-4=124386982, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-6=113137792, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-7=113210295, 
anomaly-timeline-two-minutes-error-score-changelog-5=4464429, 
anomaly-timeline-twelve-hours-error-score-changelog-7=58001249, 
anomaly-timeline-twelve-hours-error-score-changelog-5=49765578, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-one-hour-error-score-changelog-3=46566107, 
anomaly-timeline-fifteen-minutes-error-score-changelog-4=46582476, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-2=57539299, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-er
ror-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-8=4826091, 
anomaly-timeline-one-hour-error-score-changelog-7=112642671, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398, 
anomaly-timeline-twelve-hours-error-score-changelog-6=26709582}
2019-03-11 15:02:26,525 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_4] Updating store offset limit for partition 
anomaly-timeline-twelve-hours-error-score-repartition-4 to 126430132
2019-03-11 15:02:26,525 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [4_4] 
Updating store offset limits 126430132 for changelog 
anomaly-timeline-twelve-hours-error-score-repartition-4
2019-03-11 15:02:26,525 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [2_6] 
Committing
2019-03-11 15:02:26,525 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_6] Flushing all stores registered in the state manager
2019-03-11 15:02:26,525 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_6] Flushing store fifteen-minutes-error-score
2019-03-11 15:02:26,525 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_6] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020}
2019-03-11 15:02:26,528 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[2_6] Updating store offset limit for partition 
anomaly-timeline-fifteen-minutes-error-score-repartition-6 to 115827009
2019-03-11 15:02:26,528 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [2_6] 
Updating store offset limits 115827009 for changelog 
anomaly-timeline-fifteen-minutes-error-score-repartition-6
2019-03-11 15:02:26,528 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [4_5] 
Committing
2019-03-11 15:02:26,528 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_5] Flushing all stores registered in the state manager
2019-03-11 15:02:26,528 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_5] Flushing store twelve-hours-error-score
2019-03-11 15:02:26,528 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_5] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-5=4454556, 
anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,530 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[4_5] Updating store offset limit for partition 
anomaly-timeline-twelve-hours-error-score-repartition-5 to 118596513
2019-03-11 15:02:26,530 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [4_5] 
Updating store offset limits 118596513 for changelog 
anomaly-timeline-twelve-hours-error-score-repartition-5
2019-03-11 15:02:26,530 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [3_7] 
Committing
2019-03-11 15:02:26,530 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_7] Flushing all stores registered in the state manager
2019-03-11 15:02:26,531 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_7] Flushing store one-hour-error-score
2019-03-11 15:02:26,531 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_7] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-one-hour-error-score-changelog-8=9776082, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,533 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_7] Updating store offset limit for partition 
anomaly-timeline-one-hour-error-score-repartition-7 to 115248401
2019-03-11 15:02:26,533 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [3_7] 
Updating store offset limits 115248401 for changelog 
anomaly-timeline-one-hour-error-score-repartition-7
2019-03-11 15:02:26,533 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [3_8] 
Committing
2019-03-11 15:02:26,533 DEBUG 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_8] Flushing all stores registered in the state manager
2019-03-11 15:02:26,533 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_8] Flushing store one-hour-error-score
2019-03-11 15:02:26,533 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_8] Writing checkpoint: 
{anomaly-timeline-twelve-hours-error-score-changelog-4=123803165, 
anomaly-timeline-twelve-hours-error-score-changelog-3=5198411, 
anomaly-timeline-twelve-hours-error-score-changelog-1=121266870, 
anomaly-timeline-two-minutes-error-score-changelog-3=116262652, 
anomaly-timeline-twelve-hours-error-score-changelog-0=112141273, 
anomaly-timeline-two-minutes-error-score-changelog-1=121864699, 
anomaly-timeline-two-minutes-error-score-changelog-2=33179022, 
anomaly-timeline-fifteen-minutes-error-score-changelog-5=9496994, 
anomaly-timeline-one-hour-error-score-changelog-2=122882442, 
anomaly-timeline-one-hour-error-score-changelog-4=9506063, 
anomaly-timeline-fifteen-minutes-error-score-changelog-3=12114867, 
anomaly-timeline-one-hour-error-score-changelog-5=4447988, 
anomaly-timeline-fifteen-minutes-error-score-changelog-1=122051739, 
anomaly-timeline-fifteen-minutes-error-score-changelog-0=113124925, 
anomaly-timeline-one-hour-error-score-changelog-0=112251807, 
anomaly-timeline-one-hour-error-score-changelog-1=121219132, 
anomaly-timeline-fifteen-minutes-error-score-changelog-7=6933206, 
anomaly-timeline-fifteen-minutes-error-score-changelog-6=6288164, 
anomaly-timeline-two-minutes-error-score-changelog-8=5494256, 
anomaly-timeline-two-minutes-error-score-changelog-6=5472020, 
anomaly-timeline-twelve-hours-error-score-changelog-8=4815398}
2019-03-11 15:02:26,536 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.ProcessorStateManager standby-task 
[3_8] Updating store offset limit for partition 
anomaly-timeline-one-hour-error-score-repartition-8 to 112603023
2019-03-11 15:02:26,536 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.AbstractTask standby-task [3_8] 
Updating store offset limits 112603023 for changelog 
anomaly-timeline-one-hour-error-score-repartition-8
2019-03-11 15:02:26,536 TRACE 
anomaly-timeline-6f6760cc-c804-4af6-b5a2-6fa93fd6ed02-StreamThread-1 
org.apache.kafka.streams.processor.internals.StandbyTask standby-task [4_8] 
Committing{code}
I tried to debug it locally and remote, but can't reproduce it locally. I did 
have a chat with Matthias J. Stax who suggested to increase the 
state.cleanup.delay.ms but that didn't have an impact. Also because the 
downtime between restarts is just a couple of minutes. One thing I did notice 
is that just restarting one container at a time, did improve the time for the 
state store to become active. Unfortunately only for one instance.

One last thing to note: 

I did add the following code in order to override the default cleanUp on start 
and stop where the stores by default are deleted:

{{@Bean public CleanupConfig cleanupConfig() \{ return new CleanupConfig(false, 
false); }}}

So after this change, the local state dirs are left untouched when starting or 
stopping an instance. 

*Configuration*
 Confluent kafka 5.1.0 -> Kafka 2.1.0

Cluster of 3 brokers with 3 zookeepers

One topic "error-score" with 9 partititions

Streaming application using Spring Cloud Stream Kafka Binder: Fishtown.RELEASE 
-> Kafka streams 2.0.1

Configuration of stream app:
{code:java}
server.port = 5000
# Retention period is two months
windowstore.retention.ms = 5259600000

spring.jmx.enabled=false
spring.kafka.streams.application-id=anomaly-timeline
spring.kafka.streams.state-dir=./state-store
spring.cloud.stream.client-id=anomaly-timeline
spring.cloud.stream.kafka.streams.default.consumer.application-id=anomaly-timeline
spring.cloud.stream.kafka.streams.default.consumer.configuration.auto.offset.reset=earliest
spring.cloud.stream.kafka.streams.binder.configuration.commit.interval.ms=500
spring.cloud.stream.kafka.streams.binder.configuration.cache.max.bytes.buffering=2097152
spring.cloud.stream.kafka.streams.binder.configuration.num.stream.threads=1
spring.cloud.stream.kafka.streams.binder.configuration.num.standby.replicas=2
# used to broadcast the current hosts url in order to participate in 
interactive queries
spring.cloud.stream.kafka.streams.binder.configuration.application.server=localhost:5000
spring.cloud.stream.kafka.streams.binder.configuration.default.key.serde=org.apache.kafka.common.serialization.Serdes$StringSerde
# use the following timestamp extractor to deal with late arriving events
spring.cloud.stream.kafka.streams.binder.configuration.default.timestamp.extractor=ErrorScoreTimestampExtractor
spring.cloud.stream.kafka.streams.binder.configuration.default.value.serde=ErrorScoreSerde

# Spring cloud kafka defaults
spring.cloud.stream.kafka.streams.binder.brokers=localhost:9095
spring.cloud.stream.kafka.binder.auto-create-topics=false

# error-score-in binding
spring.cloud.stream.bindings.error-score-in.destination=error-score
spring.cloud.stream.bindings.error-score-in.consumer.header-mode=raw
spring.cloud.stream.bindings.error-score-in.group=anomaly-timeline
{code}
 



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to