query: 2.0.0 - (GREEN) Values are serialized as JSON
authorKai Moritz <kai@juplo.de>
Sun, 9 Jun 2024 18:44:35 +0000 (20:44 +0200)
committerKai Moritz <kai@juplo.de>
Wed, 12 Jun 2024 20:26:56 +0000 (22:26 +0200)
--
works __only__, if a default-type is defined

* The default-type is needed, to deserialized values that are read from the
  state-store.
* Without it, the deserialization fails, because not type-information is
  available.
* The type-information gets lost, when the values are stored in the state-
  store, because the message-headers are _not_ stored along with the value!

pom.xml
src/main/java/de/juplo/kafka/wordcount/query/QueryApplicationConfiguration.java
src/main/java/de/juplo/kafka/wordcount/query/QueryStreamProcessor.java
src/test/java/de/juplo/kafka/wordcount/query/QueryStreamProcessorTopologyTest.java

diff --git a/pom.xml b/pom.xml
index 60ea716..3edc6d3 100644 (file)
--- a/pom.xml
+++ b/pom.xml
                        <groupId>org.apache.kafka</groupId>
                        <artifactId>kafka-streams</artifactId>
                </dependency>
+               <dependency>
+                       <groupId>org.springframework.kafka</groupId>
+                       <artifactId>spring-kafka</artifactId>
+               </dependency>
 
                <dependency>
                        <groupId>org.springframework.boot</groupId>
                        <artifactId>spring-boot-starter-test</artifactId>
                        <scope>test</scope>
                </dependency>
-               <dependency>
-                       <groupId>org.springframework.kafka</groupId>
-                       <artifactId>spring-kafka</artifactId>
-                       <scope>test</scope>
-               </dependency>
                <dependency>
                        <groupId>org.springframework.kafka</groupId>
                        <artifactId>spring-kafka-test</artifactId>
index 50a5364..7da1712 100644 (file)
@@ -1,6 +1,5 @@
 package de.juplo.kafka.wordcount.query;
 
-import com.fasterxml.jackson.databind.ObjectMapper;
 import lombok.extern.slf4j.Slf4j;
 import org.apache.kafka.clients.consumer.ConsumerConfig;
 import org.apache.kafka.common.serialization.Serdes;
@@ -14,6 +13,8 @@ import org.springframework.boot.context.properties.EnableConfigurationProperties
 import org.springframework.context.ConfigurableApplicationContext;
 import org.springframework.context.annotation.Bean;
 import org.springframework.context.annotation.Configuration;
+import org.springframework.kafka.support.serializer.JsonDeserializer;
+import org.springframework.kafka.support.serializer.JsonSerde;
 
 import java.io.IOException;
 import java.net.InetSocketAddress;
@@ -78,7 +79,14 @@ public class QueryApplicationConfiguration
                Properties props = new Properties();
 
                props.put(StreamsConfig.DEFAULT_KEY_SERDE_CLASS_CONFIG, Serdes.String().getClass().getName());
-               props.put(StreamsConfig.DEFAULT_VALUE_SERDE_CLASS_CONFIG, Serdes.String().getClass().getName());
+               props.put(StreamsConfig.DEFAULT_VALUE_SERDE_CLASS_CONFIG, JsonSerde.class.getName());
+               props.put(JsonDeserializer.VALUE_DEFAULT_TYPE, User.class.getName()); // << Does not work without this!
+               props.put(
+                               JsonDeserializer.TYPE_MAPPINGS,
+                               "user:" + Key.class.getName() + "," +
+                               "ranking:" + Ranking.class.getName() + "," +
+                               "userdata:" + User.class.getName() + "," +
+                               "userranking:" + UserRanking.class.getName());
 
                return props;
        }
@@ -89,7 +97,6 @@ public class QueryApplicationConfiguration
                        HostInfo applicationServer,
                        QueryApplicationProperties applicationProperties,
                        KeyValueBytesStoreSupplier storeSupplier,
-                       ObjectMapper mapper,
                        ConfigurableApplicationContext context)
        {
                QueryStreamProcessor streamProcessor = new QueryStreamProcessor(
@@ -97,8 +104,7 @@ public class QueryApplicationConfiguration
                                applicationServer,
                                applicationProperties.getUsersInputTopic(),
                                applicationProperties.getRankingInputTopic(),
-                               storeSupplier,
-                               mapper);
+                               storeSupplier);
 
                streamProcessor.streams.setUncaughtExceptionHandler((Throwable e) ->
                {
index ff7c150..cc65fce 100644 (file)
@@ -1,12 +1,11 @@
 package de.juplo.kafka.wordcount.query;
 
-import com.fasterxml.jackson.core.JsonProcessingException;
-import com.fasterxml.jackson.databind.ObjectMapper;
 import jakarta.annotation.PostConstruct;
 import jakarta.annotation.PreDestroy;
 import lombok.extern.slf4j.Slf4j;
 import org.apache.kafka.common.serialization.Serdes;
 import org.apache.kafka.streams.*;
+import org.apache.kafka.streams.kstream.Consumed;
 import org.apache.kafka.streams.kstream.KStream;
 import org.apache.kafka.streams.kstream.KTable;
 import org.apache.kafka.streams.kstream.Materialized;
@@ -14,6 +13,7 @@ import org.apache.kafka.streams.state.HostInfo;
 import org.apache.kafka.streams.state.KeyValueBytesStoreSupplier;
 import org.apache.kafka.streams.state.QueryableStoreTypes;
 import org.apache.kafka.streams.state.ReadOnlyKeyValueStore;
+import org.springframework.kafka.support.serializer.JsonSerde;
 
 import java.net.URI;
 import java.util.Optional;
@@ -27,8 +27,7 @@ public class QueryStreamProcessor
 
        public final KafkaStreams streams;
        public final HostInfo hostInfo;
-       public final StoreQueryParameters<ReadOnlyKeyValueStore<String, String>> storeParameters;
-       public final ObjectMapper mapper;
+       public final StoreQueryParameters<ReadOnlyKeyValueStore<String, UserRanking>> storeParameters;
 
 
        public QueryStreamProcessor(
@@ -36,51 +35,36 @@ public class QueryStreamProcessor
                        HostInfo applicationServer,
                        String usersInputTopic,
                        String rankingInputTopic,
-                       KeyValueBytesStoreSupplier storeSupplier,
-                       ObjectMapper mapper)
+                       KeyValueBytesStoreSupplier storeSupplier)
        {
                Topology topology = buildTopology(
                                usersInputTopic,
                                rankingInputTopic,
-                               storeSupplier,
-                               mapper);
+                               storeSupplier);
                streams = new KafkaStreams(topology, props);
                hostInfo = applicationServer;
                storeParameters = StoreQueryParameters.fromNameAndType(STORE_NAME, QueryableStoreTypes.keyValueStore());;
-               this.mapper = mapper;
        }
 
        static Topology buildTopology(
                        String usersInputTopic,
                        String rankingInputTopic,
-                       KeyValueBytesStoreSupplier storeSupplier,
-                       ObjectMapper mapper)
+                       KeyValueBytesStoreSupplier storeSupplier)
        {
                StreamsBuilder builder = new StreamsBuilder();
 
-               KTable<String, String> users = builder.table(usersInputTopic);
-               KStream<String, String> rankings = builder.stream(rankingInputTopic);
+               KTable<String, User> users = builder.table(usersInputTopic);
+               KStream<String, Ranking> rankings = builder.stream(rankingInputTopic);
 
                rankings
-                               .join(users, (rankingJson, userJson) ->
-                               {
-                                       try
-                                       {
-                                               Ranking ranking = mapper.readValue(rankingJson, Ranking.class);
-                                               User user = mapper.readValue(userJson, User.class);
-
-                                               return mapper.writeValueAsString(
-                                                               UserRanking.of(
-                                                                               user.getFirstName(),
-                                                                               user.getLastName(),
-                                                                               ranking.getEntries()));
-                                       }
-                                       catch (JsonProcessingException e)
-                                       {
-                                               throw new RuntimeException(e);
-                                       }
-                               })
-                               .toTable(Materialized.as(storeSupplier));
+                               .join(users, (ranking, user) -> UserRanking.of(
+                                               user.getFirstName(),
+                                               user.getLastName(),
+                                               ranking.getEntries()))
+                               .toTable(
+                                               Materialized
+                                                               .<String, UserRanking>as(storeSupplier)
+                                                               .withValueSerde(new JsonSerde().copyWithType(UserRanking.class)));
 
                Topology topology = builder.build();
                log.info("\n\n{}", topology.describe());
@@ -111,20 +95,7 @@ public class QueryStreamProcessor
 
        public Optional<UserRanking> getUserRanking(String username)
        {
-               return
-                               Optional
-                                               .ofNullable(getStore().get(username))
-                                               .map(json ->
-                                               {
-                                                       try
-                                                       {
-                                                               return mapper.readValue(json, UserRanking.class);
-                                                       }
-                                                       catch (JsonProcessingException e)
-                                                       {
-                                                               throw new RuntimeException(e);
-                                                       }
-                                               });
+               return Optional.ofNullable(getStore().get(username));
        }
 
        @PostConstruct
index eef1eec..845792c 100644 (file)
@@ -1,6 +1,5 @@
 package de.juplo.kafka.wordcount.query;
 
-import com.fasterxml.jackson.databind.ObjectMapper;
 import de.juplo.kafka.wordcount.top10.TestRanking;
 import de.juplo.kafka.wordcount.users.TestUserData;
 import lombok.extern.slf4j.Slf4j;
@@ -39,8 +38,7 @@ public class QueryStreamProcessorTopologyTest
     Topology topology = QueryStreamProcessor.buildTopology(
         USERS_IN,
         TOP10_IN,
-        Stores.inMemoryKeyValueStore(STORE_NAME),
-        new ObjectMapper());
+        Stores.inMemoryKeyValueStore(STORE_NAME));
 
     testDriver = new TopologyTestDriver(topology, serializationConfig());