Versions Compared

Key

  • This line was added.
  • This line was removed.
  • Formatting was changed.

...

This page is meant as a template for writing a KIP. To create a KIP choose Tools->Copy on this page and modify with your content and replace the heading with the next KIP number and a description of your issue. Replace anything in italics with your own description.

Status

Current state: "Under Discussion"Adopted

Discussion thread: here 

JIRA: KAFKA-13494

Please keep the discussion on the mailing list rather than commenting on the wiki (wiki discussions get unwieldy fast).

Motivation

Provide an implementation of the Query interface, introduced in KIP-796: Interactive Query v2 , to support session and window queries.

Public Interfaces

In this KIP we propose two new public classes:

WindowKeyQuery

...

This type used to search occurrences of keys in window stores in combination with IQv2

...

. In particular the following mappings are established:

  • WindowStore.fetch(K key, Instant timeFrom, Instant timeTo) ← WindowKeyQuery.withKeyAndWindowStartRange(key, from, to)

WindowRangeQuery

The query type is used to search per window aggregates of keys in window and session stores in combination with IQv2

...

. In particular the following mappings are established:

  • WindowStore.fetchAll(Instant timeFrom, Instant timeTo) ← WindowRangeQuery.withWindowStartRange(from, to)
  • SessionStore.fetch(Bytes key) ← WindowRangeQuery.withKey(key)

There are multiple discussion points here:

  • One reason we cannot use WindowKeyQuery to support SessionStore.fetch(key) is because SessionStore.fetch(key) returns a windowed key value iterator, KeyValueIterator<Windowed<Bytes>, byte[]>, whereas WindowKeyQuery is bound to Query<WindowStoreIterator<V>>. 
  • WindowRangeQuery does not currently support key ranges. We could add more cases here to get better coverage of WindowStore and SessionStore API. In this KIP, however, we focus on a subset of queries and defer support for additional cases to future KIPs.

Proposed Changes

The WindowKeyQuery class:

Code Block
@Evolving
public class WindowKeyQuery<K, V> implements Query<WindowStoreIterator<V>> {

    

Proposed Changes

The WindowKeyQuery class:

Code Block
public class WindowKeyQuery<K, V> implements Query<WindowStoreIterator<V>> {

    private final K key;
    private final Optional<Instant> windowLower;
    private final Optional<Instant> windowUpper;

    private WindowKeyQuery(final K key, final Optional<Instant> windowLower, final Optional<Instant> windowUpper) {
        this.key = key;
        this.windowLower = windowLower;
        this.windowUpper = windowUpper;
    }

    public static <K, V> WindowKeyQuery<K, V> withKey(final K key) {
        return new WindowKeyQuery<>(key, Optional.empty(), Optional.empty());
    }

    public static <K, V> WindowKeyQuery<K, V> withKeyAndWindowBoundswithKeyAndWindowStartRange(final K key, final Instant windowLowertimeFrom, final Instant windowUpper) {
        return new WindowKeyQuery<>(key, Optional.of(windowLower), Optional.of(windowUpper));
    }timeTo);

    public K getKey() {
        return key;
    }

    public Optional<Instant> getWindowLowerBoundgetTimeFrom() {
        return windowLower;
    }

    public Optional<Instant> getWindowUpperBoundgetTimeTo() {
        return windowUpper;
    }
}

The WindowRangeQuery class:

Code Block
@Evolving
public class WindowRangeQuery<K, V> implements Query<KeyValueIterator<Windowed<K>, V>> {

    private final Optional<K> key;
    private final Optional<Instant> windowLower;
    private final Optional<Instant> windowUpper;

    private WindowRangeQuery(final Optional<K> key, final Optional<Instant> windowLower, final Optional<Instant> windowUpper) {
        this.key = key;
        this.windowLower = windowLower;
        this.windowUpper = windowUpper;
    }

    public static <K, V> WindowRangeQuery<K, V> withKey(final K key); {
         return new WindowRangeQuery<>(Optional.of(key), Optional.empty(), Optional.empty());
       }

    public static <K, V> WindowRangeQuery<K, V> withWindowRangewithWindowStartRange(final Instant windowLowertimeFrom, final Instant windowUpper) {
        return new WindowRangeQuery<>(Optional.empty(), Optional.of(windowLower), Optional.of(windowUpper));
    }timeTo);

    public Optional<K>K getKey() {
        return key;
    }

     public public Optional<Instant> getWindowLowerBoundgetTimeFrom() {
        return windowLower;
    }

    public Optional<Instant> getWindowUpperBoundgetTimeTo() {
        return windowUpper;
    }
}

Examples

The following example illustrates the use of the WindowKeyQuey class.

Code Block
Integerfinal Instant keytimeTo = 1Instant.now();
longfinal Instant lowerBoundtimeFrom = beginningOfDaytimeTo.minusSeconds(60);
 
StateQueryRequest<byte[]>final WindowKeyQuery<GenericKey, ValueAndTimestamp<GenericRow>> query = WindowKeyQuery.withKeyAndWindowStartRange(key, timeFrom, timeTo);

final StateQueryRequest<WindowStoreIterator<ValueAndTimestamp<GenericRow>>> request = inStore("rocksdbwindowstorerocksdb-window-store")
  .withQuery(WindowKeyQuery.withKey(key, lowerBound)query);
final StateQueryResult<WindowStoreIterator<ValueAndTimestamp<GenericRow>>> result     
// run the query
StateQueryResult<byte[]> result = kafkaStreams.query(query); = kafkaStream.query(request);
final WindowStoreIterator<ValueAndTimestamp<GenericRow>> iterator = result.getGlobalResult().getResult();

The following example illustrates the use of the WindowQuery class to query a window store.

Code Block
Integerfinal key1Instant = 1;
Integer key2 = 2;
long lowerBound = beginningOfDaytimeTo = Instant.now();
longfinal Instant upperBoundtimeFrom = nowtimeTo.minusSeconds(60);
 
StateQueryRequest<KeyValueIterator<Windowed<Bytes>, byte[]>>, byte[]>final WindowRangeQuery<GenericKey, ValueAndTimestamp<GenericRow>> query =
  inStore("rocksdbsessionstore")
  .withQuery(WindowQuery.withRange(key1, key2, lowerBound, upperBound));
       
// run the query
StateQueryResult<KeyValueIterator<Windowed<Bytes>>, byte[]> result = kafkaStreams.query(query);
  
// Get the results from all partitions.
final Map<Integer, QueryResult<KeyValueIterator<Windowed<Bytes>>, byte[]> partitionResults = rangeResult.getPartitionResults();
for (final Entry<Integer, QueryResult<KeyValueIterator<Windowed<Bytes>>, byte[]> entry : partitionResults.entrySet()) {
    try (final KeyValueIterator<Windowed<Bytes>>, byte[] keyValueIterator = entry.getValue().getResult()) {
        while (keyValueIterator.hasNext()) {
            final KeyValue<Bytes, byte[]> next = keyValueIterator.next();
            Bytes key = next.key.get;
            byte[] value = next.value;
        }
     }
}  WindowRangeQuery.withWindowStartRange(timeFrom, timeTo);

final StateQueryRequest<KeyValueIterator<Windowed<GenericKey>, ValueAndTimestamp<GenericRow>>> request = inStore("inmemory-window-store").withQuery(query);
final StateQueryResult<KeyValueIterator<Windowed<GenericKey>, ValueAndTimestamp<GenericRow>>> result = kafkaStreams.query(request);
final KeyValueIterator<Windowed<GenericKey>, ValueAndTimestamp<GenericRow>> iterator = result.getGlobalResult().getResult();


Compatibility, Deprecation, and Migration Plan

...