Thank you, looking forward to testing it :-)
Best regards
Patrik

> Am 06.09.2018 um 02:24 schrieb Matthias J. Sax <matth...@confluent.io>:
> 
> I create https://issues.apache.org/jira/browse/KAFKA-7380 to track this.
> 
> -Matthias
> 
>> On 8/27/18 12:07 PM, Guozhang Wang wrote:
>> Hello Patrik,
>> 
>> Thanks for the email and this is indeed a good question! :)
>> 
>> There are some historic reasons that we did the global state restoration in
>> a different way, because the restoration logic refactoring comes after
>> global KTable and global stores were introduced. But long story short, I
>> agree with you that we can indeed make global store restoration logic
>> consistent with the local state stores.
>> The tricky part is though, that global stores are shared among all
>> streaming threads within an instance (i.e. we do not keep one global task
>> per thread, but one global task per instance). So what we need to guarantee
>> is that: during a rebalance that any of the StreamThread has participated
>> (note that not all threads may participate in a single rebalance), after
>> the thread has transited to PARTITION_ASSIGNED state, we should make sure
>> the global state store is caught up to its changelog's log end offset
>> before making that threads to a RUNNING state.
>> 
>> Some initial thoughts about how to do it:
>> 
>> 1. Whenever a thread transited to PARTITION_ASSIGNED state, check if the
>> global store's restoration has completed; if not, halt on further
>> transiting to RUNNING, and tries to start restoring global stores.
>> 2. Only start a task after its corresponding stores changelog has been
>> restored, IN ADDITION, the global stores being restored as well.
>> 
>> If you like, please feel free to create a JIRA requesting this improvement
>> so someone can work on it someday.
>> 
>> Guozhang
>> 
>> 
>> 
>> 
>>> On Sat, Aug 25, 2018 at 10:44 AM, Patrik Kleindl <pklei...@gmail.com> wrote:
>>> 
>>> Hello
>>> 
>>> We are currently using GlobalKTables for interactive queries as well as for
>>> lookups inside stream applications but have come across some
>>> limitations/problems.
>>> The main problem was that our deployments including application start took
>>> longer with every new global state store we added which cause some
>>> operational issues because a timeout of 10 minutes was reached.
>>> As this was not expected from the documentation I investigated a little:
>>> According to the documentation the call the KafkaStreams start method
>>> should always return immediately and not block the calling thread.
>>> 
>>> *Start the KafkaStreams instance by starting all its threads. This function
>>> is expected to be called only once during the life cycle of the
>>> client.**Because
>>> threads are started in the background, this method does not block.*
>>> 
>>> But if one (or more) GlobalStateStores are initialized then this call will
>>> take a considerable amount of time and block.
>>> This happens because the GlobalStreamThread only changes to running after
>>> initialize() is done which does all the state restore etc in loops.
>>> An inquiry to the support yielded the answer that this was working as
>>> designed and the documentation will be updated.
>>> 
>>> While we have worked around the issue by asynchronously calling the start
>>> method the question remains why this is intended?
>>> 
>>> If I understand https://issues.apache.org/jira/browse/KAFKA-6205 correctly
>>> the topology should not be initialized before the state stores are
>>> restored, so why is it necessary to handle global state stores differently
>>> than local ones in this aspect?
>>> Additionally, for global state stores all stores and partitions are
>>> initialized sequentially while local state stores are always handled in
>>> parallel (per store and for all partitions)
>>> 
>>> Any help is welcome, maybe it would be worth a KIP to improve this
>>> situation.
>>> A code sample as well as logs for both cases can be found below.
>>> 
>>> best regards
>>> 
>>> Patrik
>>> 
>>> PS: Sample code to reproduce, just switch between GlobalKTable and KTable,
>>> the topic used should have 10+ partitions and contain a few 100K records to
>>> show some meaningful results:
>>> 
>>> import org.apache.kafka.common.serialization.Serdes;
>>> import org.apache.kafka.streams.KafkaStreams;
>>> import org.apache.kafka.streams.StreamsBuilder;
>>> import org.apache.kafka.streams.StreamsConfig;
>>> import org.apache.kafka.streams.kstream.GlobalKTable;
>>> import org.apache.kafka.streams.kstream.KTable;
>>> import org.apache.kafka.streams.kstream.Materialized;
>>> 
>>> import java.util.Date;
>>> import java.util.Properties;
>>> 
>>> public class TestTables {
>>> 
>>>    public static void main(String[] args) {
>>>        final String TOPIC_NAME = "testtables";
>>> 
>>>        Properties streamsConfiguration = new Properties();
>>>        streamsConfiguration.put(StreamsConfig.APPLICATION_ID_CONFIG,
>>> "testtables");
>>>        streamsConfiguration.put(StreamsConfig.BOOTSTRAP_SERVERS_CONFIG,
>>> "broker0:9092");
>>> 
>>>        StreamsBuilder builder = new StreamsBuilder();
>>> 
>>>        GlobalKTable<String, String> testtable =
>>> builder.globalTable(TOPIC_NAME, Materialized.with(Serdes.String(),
>>> Serdes.String()));
>>>        //KTable<String, String> testtable = builder.table(TOPIC_NAME,
>>> Materialized.with(Serdes.String(), Serdes.String()));
>>> 
>>>        final KafkaStreams streams = new KafkaStreams(builder.build(),
>>> streamsConfiguration);
>>> 
>>>        streams.cleanUp();
>>>        streams.setStateListener((state, state1) -> {
>>>            if (state == KafkaStreams.State.RUNNING && state1 ==
>>> KafkaStreams.State.REBALANCING)
>>>                System.out.println("Running " + new Date());
>>>        });
>>>        System.out.println("Starting " + new Date());
>>>        streams.start();
>>>        System.out.println("Started " + new Date());
>>> 
>>>        Runtime.getRuntime().addShutdownHook(new Thread(() -> {
>>>            try {
>>>                streams.close();
>>>            } catch (Exception e) {
>>>                // ignored
>>>            }
>>>        }));
>>> 
>>>    }
>>> }
>>> 
>>> Log for KTable:
>>> Starting Fri Aug 24 20:17:19 CEST 2018
>>> 
>>> 2018-08-24 20:17:19 DEBUG KafkaStreams:759 - stream-client
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7] Starting Streams client
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamThread:713 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Starting
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamThread:200 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] State
>>> transition from CREATED to RUNNING
>>> 
>>> 2018-08-24 20:17:19 INFO  KafkaStreams:782 - stream-client
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7] Started Streams client
>>> 
>>> Started Fri Aug 24 20:17:19 CEST 2018
>>> 
>>> 2018-08-24 20:17:19 INFO  Metadata:265 - Cluster ID: 6YtMshjsT_WQNwRGhTyc4A
>>> 
>>> 2018-08-24 20:17:19 INFO  AbstractCoordinator:605 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Discovered group coordinator broker2:9292 (id:
>>> 2147483645 rack: null)
>>> 
>>> 2018-08-24 20:17:19 INFO  ConsumerCoordinator:411 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Revoking previously assigned partitions []
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamThread:279 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] at state
>>> RUNNING: partitions [] revoked at the beginning of consumer rebalance.
>>> 
>>> current assigned active tasks: []
>>> 
>>> current assigned standby tasks: []
>>> 
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamThread:200 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] State
>>> transition from RUNNING to PARTITIONS_REVOKED
>>> 
>>> 2018-08-24 20:17:19 INFO  KafkaStreams:261 - stream-client
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7] State transition from
>>> RUNNING to REBALANCING
>>> 
>>> 2018-08-24 20:17:19 DEBUG TaskManager:238 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Suspending
>>> all active tasks [] and standby tasks []
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamThread:299 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] partition
>>> revocation took 1 ms.
>>> 
>>> suspended active tasks: []
>>> 
>>> suspended standby tasks: []
>>> 
>>> 2018-08-24 20:17:19 INFO  AbstractCoordinator:442 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-consumer,
>>> groupId=testtables] (Re-)joining group
>>> 
>>> 2018-08-24 20:17:19 DEBUG InternalTopologyBuilder:1881 - [testtables]found
>>> stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]  topics
>>> possibly matching regex
>>> 
>>> 2018-08-24 20:17:19 DEBUG InternalTopologyBuilder:1275 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] updating
>>> builder with SubscriptionUpdates{updatedTopicSubscriptions=[testtables]}
>>> topic(s) with possible matching regex subscription(s)
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:302 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Constructed client metadata
>>> {8ca25259-6bb2-4bc1-9293-fc05e1b28ce7=ClientMetadata{hostInfo=null,
>>> consumers=[testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-
>>> consumer-f39a265f-527a-47a3-a161-29892a4b7a94],
>>> state=[activeTasks: ([]) standbyTasks: ([]) assignedTasks: ([])
>>> prevActiveTasks: ([]) prevAssignedTasks: ([]) capacity: 1]}} from the
>>> member subscriptions.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:634 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Starting to validate internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:658 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Completed validating internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:391 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Created repartition topics [] from the parsed topology.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:634 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Starting to validate internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:17:19 DEBUG InternalTopicManager:201 - stream-thread [main]
>>> Could not get number of partitions for topic
>>> testtables-testtables-STATE-STORE-0000000000-changelog.
>>> 
>>> 2018-08-24 20:17:19 DEBUG InternalTopicManager:98 - stream-thread [main]
>>> Going to create topic
>>> testtables-testtables-STATE-STORE-0000000000-changelog with 12 partitions
>>> and config {cleanup.policy=compact}.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:658 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Completed validating internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:465 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Created state changelog topics
>>> [InternalTopicMetadata(config=UnwindowedChangelogTopicConfig
>>> (name=testtables-testtables-STATE-STORE-0000000000-changelog,
>>> topicConfigs={}), numPartitions=12)] from the parsed topology.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamPartitionAssignor:475 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Assigning tasks [0_0, 0_1, 0_2, 0_3, 0_4, 0_5, 0_6, 0_7, 0_8, 0_9, 0_10,
>>> 0_11] to clients {8ca25259-6bb2-4bc1-9293-fc05e1b28ce7=[activeTasks: ([])
>>> standbyTasks: ([]) assignedTasks: ([]) prevActiveTasks: ([])
>>> prevAssignedTasks: ([]) capacity: 1]} with number of replicas 0
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamPartitionAssignor:481 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1-consumer]
>>> Assigned tasks to clients as
>>> {8ca25259-6bb2-4bc1-9293-fc05e1b28ce7=[activeTasks: ([0_0, 0_1, 0_2, 0_3,
>>> 0_4, 0_5, 0_6, 0_7, 0_8, 0_9, 0_10, 0_11]) standbyTasks: ([])
>>> assignedTasks: ([0_0, 0_1, 0_2, 0_3, 0_4, 0_5, 0_6, 0_7, 0_8, 0_9, 0_10,
>>> 0_11]) prevActiveTasks: ([]) prevAssignedTasks: ([]) capacity: 1]}.
>>> 
>>> 2018-08-24 20:17:19 INFO  AbstractCoordinator:409 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Successfully joined group with generation 9
>>> 
>>> 2018-08-24 20:17:19 INFO  ConsumerCoordinator:256 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Setting newly assigned partitions [testtables-0,
>>> testtables-8, testtables-7, testtables-6, testtables-5, testtables-4,
>>> testtables-3, testtables-2, testtables-1, testtables-11, testtables-10,
>>> testtables-9]
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamThread:247 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] at state
>>> PARTITIONS_REVOKED: partitions [testtables-0, testtables-8, testtables-7,
>>> testtables-6, testtables-5, testtables-4, testtables-3, testtables-2,
>>> testtables-1, testtables-11, testtables-10, testtables-9] assigned at the
>>> end of consumer rebalance.
>>> 
>>> current suspended active tasks: []
>>> 
>>> current suspended standby tasks: []
>>> 
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamThread:200 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] State
>>> transition from PARTITIONS_REVOKED to PARTITIONS_ASSIGNED
>>> 
>>> 2018-08-24 20:17:19 DEBUG TaskManager:120 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Adding
>>> assigned tasks as active: {0_0=[testtables-0], 0_1=[testtables-1],
>>> 0_2=[testtables-2], 0_3=[testtables-3], 0_4=[testtables-4],
>>> 0_5=[testtables-5], 0_6=[testtables-6], 0_7=[testtables-7],
>>> 0_8=[testtables-8], 0_9=[testtables-9], 0_10=[testtables-10],
>>> 0_11=[testtables-11]}
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_0] Created
>>> state store manager for task 0_0 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_0] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_1] Created
>>> state store manager for task 0_1 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_1] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_2] Created
>>> state store manager for task 0_2 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_2] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_3] Created
>>> state store manager for task 0_3 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_3] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_4] Created
>>> state store manager for task 0_4 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_4] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_5] Created
>>> state store manager for task 0_5 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_5] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_6] Created
>>> state store manager for task 0_6 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_6] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_7] Created
>>> state store manager for task 0_7 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_7] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_8] Created
>>> state store manager for task 0_8 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_8] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_9] Created
>>> state store manager for task 0_9 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_9] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_10] Created
>>> state store manager for task 0_10 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_10] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:96 - task [0_11] Created
>>> state store manager for task 0_11 with the acquired state dir lock
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:332 - task [0_11] Register
>>> global stores []
>>> 
>>> 2018-08-24 20:17:19 INFO  StreamThread:266 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] partition
>>> assignment took 28 ms.
>>> 
>>> current active tasks: [0_0, 0_1, 0_2, 0_3, 0_4, 0_5, 0_6, 0_7, 0_8, 0_9,
>>> 0_10, 0_11]
>>> 
>>> current standby tasks: []
>>> 
>>> previous active tasks: []
>>> 
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:83 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Initializing stream tasks [0_0, 0_1, 0_2, 0_3, 0_4, 0_5, 0_6, 0_7, 0_8,
>>> 0_9, 0_10, 0_11]
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_0
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_0]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_0 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_1
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_1]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_1 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_2
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_2]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_2 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_3
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_3]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_3 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_4
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_4]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_4 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_5
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_5]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_5 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_6
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_6]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_6 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_7
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_7]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_7 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_8
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_8]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_8 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_9
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_9]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_9 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_10
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_10]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_10 to restoring
>>> 
>>> 2018-08-24 20:17:19 DEBUG StateDirectory:163 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Acquired
>>> state dir lock for task 0_11
>>> 
>>> 2018-08-24 20:17:19 DEBUG ProcessorStateManager:113 - task [0_11]
>>> Registering state store testtables-STATE-STORE-0000000000 to its state
>>> manager
>>> 
>>> 2018-08-24 20:17:19 DEBUG AssignedStreamsTasks:89 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> Transitioning stream task 0_11 to restoring
>>> 
>>> 2018-08-24 20:17:19 INFO  Metadata:265 - Cluster ID: 6YtMshjsT_WQNwRGhTyc4A
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:165 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Start
>>> restoring state stores from changelog topics [testtables-0, testtables-11,
>>> testtables-10, testtables-9, testtables-8, testtables-7, testtables-6,
>>> testtables-5, testtables-4, testtables-3, testtables-2, testtables-1]
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-10 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-7 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-4 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-1 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-11 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-8 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-5 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-2 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-0 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-9 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-6 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-
>>> StreamThread-1-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-3 to offset 0.
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-0 from offset 0 to endOffset 41766
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-11 from offset 0 to endOffset 67870
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-10 from offset 0 to endOffset 62880
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-9 from offset 0 to endOffset 66798
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-8 from offset 0 to endOffset 62851
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-7 from offset 0 to endOffset 66015
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-6 from offset 0 to endOffset 68018
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-5 from offset 0 to endOffset 61515
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-4 from offset 0 to endOffset 39368
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-3 from offset 0 to endOffset 68995
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-2 from offset 0 to endOffset 38719
>>> 
>>> 2018-08-24 20:17:19 DEBUG StoreChangelogReader:201 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Restoring
>>> partition testtables-1 from offset 0 to endOffset 62722
>>> 
>>> 2018-08-24 20:17:19 DEBUG StreamThread:1007 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Committed
>>> all active tasks [0_0, 0_1, 0_2, 0_3, 0_4, 0_5, 0_6, 0_7, 0_8, 0_9, 0_10,
>>> 0_11] and standby tasks [] in 0ms
>>> 
>>> 2018-08-24 20:17:26 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-5 with 34769 records ranging from
>>> offset 0 to 34769
>>> 
>>> 2018-08-24 20:17:26 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-2 with 30916 records ranging from
>>> offset 0 to 30916
>>> 
>>> 2018-08-24 20:17:26 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_2 to running
>>> 
>>> 2018-08-24 20:17:26 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_5 to running
>>> 
>>> 2018-08-24 20:17:26 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-11 with 37275 records ranging
>>> from offset 0 to 37275
>>> 
>>> 2018-08-24 20:17:26 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_11 to running
>>> 
>>> 2018-08-24 20:17:26 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-8 with 37177 records ranging from
>>> offset 0 to 37177
>>> 
>>> 2018-08-24 20:17:26 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_8 to running
>>> 
>>> 2018-08-24 20:17:28 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-1 with 37028 records ranging from
>>> offset 0 to 37028
>>> 
>>> 2018-08-24 20:17:28 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_1 to running
>>> 
>>> 2018-08-24 20:17:28 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-4 with 32595 records ranging from
>>> offset 0 to 32595
>>> 
>>> 2018-08-24 20:17:28 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_4 to running
>>> 
>>> 2018-08-24 20:17:29 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-10 with 36822 records ranging
>>> from offset 0 to 36822
>>> 
>>> 2018-08-24 20:17:29 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-7 with 36723 records ranging from
>>> offset 0 to 36723
>>> 
>>> 2018-08-24 20:17:29 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_7 to running
>>> 
>>> 2018-08-24 20:17:29 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_10 to running
>>> 
>>> 2018-08-24 20:17:29 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-3 with 36600 records ranging from
>>> offset 0 to 36600
>>> 
>>> 2018-08-24 20:17:29 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_3 to running
>>> 
>>> 2018-08-24 20:17:29 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-9 with 36169 records ranging from
>>> offset 0 to 36169
>>> 
>>> 2018-08-24 20:17:29 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_9 to running
>>> 
>>> 2018-08-24 20:17:29 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-6 with 35650 records ranging from
>>> offset 0 to 35650
>>> 
>>> 2018-08-24 20:17:29 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_6 to running
>>> 
>>> 2018-08-24 20:17:29 DEBUG StoreChangelogReader:267 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] Completed
>>> restoring state from changelog testtables-0 with 32356 records ranging from
>>> offset 0 to 32356
>>> 
>>> 2018-08-24 20:17:29 DEBUG AssignedStreamsTasks:264 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1]
>>> transitioning stream task 0_0 to running
>>> 
>>> 2018-08-24 20:17:29 INFO  StreamThread:200 - stream-thread
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7-StreamThread-1] State
>>> transition from PARTITIONS_ASSIGNED to RUNNING
>>> 
>>> 2018-08-24 20:17:29 INFO  KafkaStreams:261 - stream-client
>>> [testtables-8ca25259-6bb2-4bc1-9293-fc05e1b28ce7] State transition from
>>> REBALANCING to RUNNING
>>> 
>>> Running Fri Aug 24 20:17:29 CEST 2018
>>> 
>>> 
>>> Log for GlobalKTable:
>>> 
>>> Starting Fri Aug 24 20:19:04 CEST 2018
>>> 
>>> 2018-08-24 20:19:04 DEBUG KafkaStreams:759 - stream-client
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086] Starting Streams client
>>> 
>>> 2018-08-24 20:19:04 DEBUG StateDirectory:192 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-GlobalStreamThread]
>>> Acquired global state dir lock
>>> 
>>> 2018-08-24 20:19:04 INFO  Metadata:265 - Cluster ID: 6YtMshjsT_WQNwRGhTyc4A
>>> 
>>> 2018-08-24 20:19:04 INFO  GlobalStateManagerImpl:158 - global-stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-GlobalStreamThread]
>>> Restoring state for global store testtables-STATE-STORE-0000000000
>>> 
>>> 2018-08-24 20:19:04 INFO  Metadata:265 - Cluster ID: 6YtMshjsT_WQNwRGhTyc4A
>>> 
>>> 2018-08-24 20:19:04 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-8 to offset 0.
>>> 
>>> 2018-08-24 20:19:07 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-2 to offset 0.
>>> 
>>> 2018-08-24 20:19:10 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-11 to offset 0.
>>> 
>>> 2018-08-24 20:19:12 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-5 to offset 0.
>>> 
>>> 2018-08-24 20:19:15 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-4 to offset 0.
>>> 
>>> 2018-08-24 20:19:18 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-7 to offset 0.
>>> 
>>> 2018-08-24 20:19:20 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-1 to offset 0.
>>> 
>>> 2018-08-24 20:19:22 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-10 to offset 0.
>>> 
>>> 2018-08-24 20:19:24 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-9 to offset 0.
>>> 
>>> 2018-08-24 20:19:27 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-3 to offset 0.
>>> 
>>> 2018-08-24 20:19:33 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-6 to offset 0.
>>> 
>>> 2018-08-24 20:19:37 INFO  Fetcher:561 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> global-restore-consumer,
>>> groupId=] Resetting offset for partition testtables-0 to offset 0.
>>> 
>>> 2018-08-24 20:19:39 INFO  GlobalStreamThread:157 - global-stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-GlobalStreamThread] State
>>> transition from CREATED to RUNNING
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:713 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] Starting
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:200 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] State
>>> transition from CREATED to RUNNING
>>> 
>>> 2018-08-24 20:19:39 INFO  KafkaStreams:782 - stream-client
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086] Started Streams client
>>> 
>>> Started Fri Aug 24 20:19:39 CEST 2018
>>> 
>>> 2018-08-24 20:19:39 INFO  Metadata:265 - Cluster ID: 6YtMshjsT_WQNwRGhTyc4A
>>> 
>>> 2018-08-24 20:19:39 INFO  AbstractCoordinator:605 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Discovered group coordinator broker2:9292 (id:
>>> 2147483645 rack: null)
>>> 
>>> 2018-08-24 20:19:39 INFO  ConsumerCoordinator:411 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Revoking previously assigned partitions []
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamThread:279 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] at state
>>> RUNNING: partitions [] revoked at the beginning of consumer rebalance.
>>> 
>>> current assigned active tasks: []
>>> 
>>> current assigned standby tasks: []
>>> 
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:200 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] State
>>> transition from RUNNING to PARTITIONS_REVOKED
>>> 
>>> 2018-08-24 20:19:39 INFO  KafkaStreams:261 - stream-client
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086] State transition from
>>> RUNNING to REBALANCING
>>> 
>>> 2018-08-24 20:19:39 DEBUG TaskManager:238 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1]
>>> Suspending
>>> all active tasks [] and standby tasks []
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:299 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] partition
>>> revocation took 0 ms.
>>> 
>>> suspended active tasks: []
>>> 
>>> suspended standby tasks: []
>>> 
>>> 2018-08-24 20:19:39 INFO  AbstractCoordinator:442 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> StreamThread-1-consumer,
>>> groupId=testtables] (Re-)joining group
>>> 
>>> 2018-08-24 20:19:39 DEBUG InternalTopologyBuilder:1881 - [testtables]found
>>> stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1]  topics
>>> possibly matching regex
>>> 
>>> 2018-08-24 20:19:39 DEBUG InternalTopologyBuilder:1275 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] updating
>>> builder with SubscriptionUpdates{updatedTopicSubscriptions=[testtables]}
>>> topic(s) with possible matching regex subscription(s)
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:302 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Constructed client metadata
>>> {cb808539-c805-4cd8-8467-4ebf618bb086=ClientMetadata{hostInfo=null,
>>> consumers=[testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-
>>> consumer-a82cbb2b-172b-423c-86fb-6ef01b3bf2d2],
>>> state=[activeTasks: ([]) standbyTasks: ([]) assignedTasks: ([])
>>> prevActiveTasks: ([]) prevAssignedTasks: ([]) capacity: 1]}} from the
>>> member subscriptions.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:634 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Starting to validate internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:658 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Completed validating internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:391 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Created repartition topics [] from the parsed topology.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:634 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Starting to validate internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:658 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Completed validating internal topics in partition assignor.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:465 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Created state changelog topics [] from the parsed topology.
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamPartitionAssignor:475 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Assigning tasks [] to clients
>>> {cb808539-c805-4cd8-8467-4ebf618bb086=[activeTasks: ([]) standbyTasks:
>>> ([])
>>> assignedTasks: ([]) prevActiveTasks: ([]) prevAssignedTasks: ([]) capacity:
>>> 1]} with number of replicas 0
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamPartitionAssignor:481 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1-consumer]
>>> Assigned tasks to clients as
>>> {cb808539-c805-4cd8-8467-4ebf618bb086=[activeTasks: ([]) standbyTasks:
>>> ([])
>>> assignedTasks: ([]) prevActiveTasks: ([]) prevAssignedTasks: ([]) capacity:
>>> 1]}.
>>> 
>>> 2018-08-24 20:19:39 WARN  ConsumerCoordinator:376 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> StreamThread-1-consumer,
>>> groupId=testtables] The following subscribed topics are not assigned to any
>>> members: [testtables]
>>> 
>>> 2018-08-24 20:19:39 INFO  AbstractCoordinator:409 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Successfully joined group with generation 11
>>> 
>>> 2018-08-24 20:19:39 INFO  ConsumerCoordinator:256 - [Consumer
>>> clientId=testtables-cb808539-c805-4cd8-8467-4ebf618bb086-
>>> StreamThread-1-consumer,
>>> groupId=testtables] Setting newly assigned partitions []
>>> 
>>> 2018-08-24 20:19:39 DEBUG StreamThread:247 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] at state
>>> PARTITIONS_REVOKED: partitions [] assigned at the end of consumer
>>> rebalance.
>>> 
>>> current suspended active tasks: []
>>> 
>>> current suspended standby tasks: []
>>> 
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:200 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] State
>>> transition from PARTITIONS_REVOKED to PARTITIONS_ASSIGNED
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:266 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] partition
>>> assignment took 0 ms.
>>> 
>>> current active tasks: []
>>> 
>>> current standby tasks: []
>>> 
>>> previous active tasks: []
>>> 
>>> 
>>> 2018-08-24 20:19:39 INFO  StreamThread:200 - stream-thread
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086-StreamThread-1] State
>>> transition from PARTITIONS_ASSIGNED to RUNNING
>>> 
>>> 2018-08-24 20:19:39 INFO  KafkaStreams:261 - stream-client
>>> [testtables-cb808539-c805-4cd8-8467-4ebf618bb086] State transition from
>>> REBALANCING to RUNNING
>>> 
>>> Running Fri Aug 24 20:19:39 CEST 2018
>>> 
>> 
>> 
>> 
> 

Reply via email to