You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by tr...@apache.org on 2022/02/07 15:55:08 UTC

[flink-statefun-playground] branch release-3.2 updated (8f0149e -> 54fb18d)

This is an automated email from the ASF dual-hosted git repository.

trohrmann pushed a change to branch release-3.2
in repository https://gitbox.apache.org/repos/asf/flink-statefun-playground.git.


    from 8f0149e  [hotfix] typo backquote
     new e956104  [FLINK-25935] Add LocalEnvironmentEntrypoint
     new 1529f87  [FLINK-25934] Use apache/flink-statefun-playground image in docker-compose.ymls
     new d1fa6dd  [FLINK-25934] Replace Kafka image with Redpanda
     new 54fb18d  [FLINK-25934] Reduce Statefun's memory consumption to 300mb

The 4 revisions listed above as "new" are entirely new to this
repository and will be described in separate emails.  The revisions
listed as "add" were already present in the repository and have only
been added to this reference.


Summary of changes:
 go/greeter/README.md                               |  14 +--
 go/greeter/docker-compose.yml                      |  56 ++++-------
 go/showcase/docker-compose.yml                     |  54 ++++------
 java/connected-components/README.md                |   5 +-
 java/connected-components/docker-compose.yml       |  63 ++++--------
 java/greeter/README.md                             |   5 +-
 java/greeter/docker-compose.yml                    |  63 ++++--------
 java/shopping-cart/README.md                       |   2 +-
 java/shopping-cart/docker-compose.yml              |  59 ++++-------
 javascript/greeter/README.md                       |  14 +--
 javascript/greeter/docker-compose.yml              |  56 ++++-------
 javascript/showcase/docker-compose.yml             |  54 ++++------
 .../statefun-playground-entrypoint}/Dockerfile     |   9 +-
 .../statefun-playground-entrypoint/README.md       |  12 +++
 .../docker-entrypoint.sh                           |  10 ++
 .../statefun-playground-entrypoint}/pom.xml        | 111 +++++++++++----------
 .../entrypoint/ClassPathUniverseProvider.java      |  40 ++++++++
 .../entrypoint/ConfigurationValidator.java         |  85 ++++++++++++++++
 .../entrypoint/LocalEnvironmentEntrypoint.java     | 108 ++++++++++++++++++++
 .../src/main/resources/log4j.properties            |  22 +---
 python/greeter/README.md                           |  14 +--
 python/greeter/docker-compose.yml                  |  56 ++++-------
 python/showcase/docker-compose.yml                 |  54 ++++------
 23 files changed, 517 insertions(+), 449 deletions(-)
 copy {java/connected-components => playground-internal/statefun-playground-entrypoint}/Dockerfile (81%)
 create mode 100644 playground-internal/statefun-playground-entrypoint/README.md
 create mode 100644 playground-internal/statefun-playground-entrypoint/docker-entrypoint.sh
 copy {java/greeter => playground-internal/statefun-playground-entrypoint}/pom.xml (51%)
 create mode 100644 playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ClassPathUniverseProvider.java
 create mode 100644 playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ConfigurationValidator.java
 create mode 100644 playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java
 copy go/greeter/Dockerfile => playground-internal/statefun-playground-entrypoint/src/main/resources/log4j.properties (76%)

[flink-statefun-playground] 02/04: [FLINK-25934] Use apache/flink-statefun-playground image in docker-compose.ymls

Posted by tr...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

trohrmann pushed a commit to branch release-3.2
in repository https://gitbox.apache.org/repos/asf/flink-statefun-playground.git

commit 1529f87048fba52c9f287bbfa002abc4d935e0ea
Author: Till Rohrmann <tr...@apache.org>
AuthorDate: Fri Feb 4 19:08:54 2022 +0100

    [FLINK-25934] Use apache/flink-statefun-playground image in docker-compose.ymls
---
 go/greeter/docker-compose.yml                | 26 +++++-----------------
 go/showcase/docker-compose.yml               | 24 +++++---------------
 java/connected-components/docker-compose.yml | 33 ++++------------------------
 java/greeter/docker-compose.yml              | 31 ++++----------------------
 java/shopping-cart/docker-compose.yml        | 29 +++---------------------
 javascript/greeter/docker-compose.yml        | 26 +++++-----------------
 javascript/showcase/docker-compose.yml       | 24 +++++---------------
 python/greeter/docker-compose.yml            | 26 +++++-----------------
 python/showcase/docker-compose.yml           | 24 +++++---------------
 9 files changed, 41 insertions(+), 202 deletions(-)

diff --git a/go/greeter/docker-compose.yml b/go/greeter/docker-compose.yml
index dc86400..f72a966 100644
--- a/go/greeter/docker-compose.yml
+++ b/go/greeter/docker-compose.yml
@@ -33,31 +33,15 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0
-    expose:
-      - "6121"
-      - "6122"
     depends_on:
-      - statefun-manager
       - kafka
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
+      - functions
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
@@ -90,7 +74,7 @@ services:
     image: ververica/statefun-playground-producer:latest
     depends_on:
       - kafka
-      - statefun-worker
+      - statefun
     environment:
       APP_PATH: /mnt/input-example.json
       APP_KAFKA_HOST: kafka:9092
diff --git a/go/showcase/docker-compose.yml b/go/showcase/docker-compose.yml
index eb5c6c9..f91c0a1 100644
--- a/go/showcase/docker-compose.yml
+++ b/go/showcase/docker-compose.yml
@@ -23,28 +23,14 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0
-    expose:
-      - "6121"
-      - "6122"
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
+    depends_on:
+      - kafka
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
diff --git a/java/connected-components/docker-compose.yml b/java/connected-components/docker-compose.yml
index 9402020..d66fc1f 100644
--- a/java/connected-components/docker-compose.yml
+++ b/java/connected-components/docker-compose.yml
@@ -34,36 +34,15 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/connected-components/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
     depends_on:
-      - statefun-manager
       - kafka
       - connected-components-functions
-    links:
-      - "statefun-manager:statefun-manager"
-      - "kafka:kafka"
-      - "connected-components-functions:connected-components-functions"
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
     volumes:
-      - ./module.yaml:/opt/statefun/modules/connected-components/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
@@ -82,8 +61,6 @@ services:
       - "9092:9092"
     depends_on:
       - zookeeper
-    links:
-      - "zookeeper:zookeeper"
     environment:
       KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
       KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
@@ -98,9 +75,7 @@ services:
     image: ververica/statefun-playground-producer:latest
     depends_on:
       - kafka
-      - statefun-worker
-    links:
-      - "kafka:kafka"
+      - statefun
     environment:
       APP_PATH: /mnt/vertices.txt
       APP_KAFKA_HOST: kafka:9092
diff --git a/java/greeter/docker-compose.yml b/java/greeter/docker-compose.yml
index c863641..fec846d 100644
--- a/java/greeter/docker-compose.yml
+++ b/java/greeter/docker-compose.yml
@@ -34,36 +34,15 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
     depends_on:
-      - statefun-manager
       - kafka
       - greeter-functions
-    links:
-      - "statefun-manager:statefun-manager"
-      - "kafka:kafka"
-      - "greeter-functions:greeter-functions"
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
@@ -98,9 +77,7 @@ services:
     image: ververica/statefun-playground-producer:latest
     depends_on:
       - kafka
-      - statefun-worker
-    links:
-      - "kafka:kafka"
+      - statefun
     environment:
       APP_PATH: /mnt/user-logins.txt
       APP_KAFKA_HOST: kafka:9092
diff --git a/java/shopping-cart/docker-compose.yml b/java/shopping-cart/docker-compose.yml
index 090f09e..3a2d25e 100644
--- a/java/shopping-cart/docker-compose.yml
+++ b/java/shopping-cart/docker-compose.yml
@@ -34,36 +34,15 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/shopping-cart/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
     depends_on:
-      - statefun-manager
       - kafka
       - shopping-cart-functions
-    links:
-      - "statefun-manager:statefun-manager"
-      - "kafka:kafka"
-      - "shopping-cart-functions:shopping-cart-functions"
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
     volumes:
-      - ./module.yaml:/opt/statefun/modules/shopping-cart/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
@@ -82,8 +61,6 @@ services:
       - "9092:9092"
     depends_on:
       - zookeeper
-    links:
-      - "zookeeper:zookeeper"
     environment:
       KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
       KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
diff --git a/javascript/greeter/docker-compose.yml b/javascript/greeter/docker-compose.yml
index 583a35c..f72a966 100644
--- a/javascript/greeter/docker-compose.yml
+++ b/javascript/greeter/docker-compose.yml
@@ -33,31 +33,15 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
     depends_on:
-      - statefun-manager
       - kafka
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
+      - functions
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
@@ -90,7 +74,7 @@ services:
     image: ververica/statefun-playground-producer:latest
     depends_on:
       - kafka
-      - statefun-worker
+      - statefun
     environment:
       APP_PATH: /mnt/input-example.json
       APP_KAFKA_HOST: kafka:9092
diff --git a/javascript/showcase/docker-compose.yml b/javascript/showcase/docker-compose.yml
index 09b43a4..e07f6d6 100644
--- a/javascript/showcase/docker-compose.yml
+++ b/javascript/showcase/docker-compose.yml
@@ -23,28 +23,14 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
+    depends_on:
+      - kafka
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
diff --git a/python/greeter/docker-compose.yml b/python/greeter/docker-compose.yml
index 583a35c..f72a966 100644
--- a/python/greeter/docker-compose.yml
+++ b/python/greeter/docker-compose.yml
@@ -33,31 +33,15 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
     depends_on:
-      - statefun-manager
       - kafka
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
+      - functions
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress
@@ -90,7 +74,7 @@ services:
     image: ververica/statefun-playground-producer:latest
     depends_on:
       - kafka
-      - statefun-worker
+      - statefun
     environment:
       APP_PATH: /mnt/input-example.json
       APP_KAFKA_HOST: kafka:9092
diff --git a/python/showcase/docker-compose.yml b/python/showcase/docker-compose.yml
index 09b43a4..e07f6d6 100644
--- a/python/showcase/docker-compose.yml
+++ b/python/showcase/docker-compose.yml
@@ -23,28 +23,14 @@ services:
   #    StateFun runtime
   ###############################################################
 
-  statefun-manager:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6123"
+  statefun:
+    image: apache/flink-statefun-playground:3.2.0
     ports:
       - "8081:8081"
-    environment:
-      ROLE: master
-      MASTER_HOST: statefun-manager
-    volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
-
-  statefun-worker:
-    image: apache/flink-statefun:3.2.0-java11
-    expose:
-      - "6121"
-      - "6122"
-    environment:
-      ROLE: worker
-      MASTER_HOST: statefun-manager
+    depends_on:
+      - kafka
     volumes:
-      - ./module.yaml:/opt/statefun/modules/greeter/module.yaml
+      - ./module.yaml:/module.yaml
 
   ###############################################################
   #    Kafka for ingress and egress

[flink-statefun-playground] 04/04: [FLINK-25934] Reduce Statefun's memory consumption to 300mb

Posted by tr...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

trohrmann pushed a commit to branch release-3.2
in repository https://gitbox.apache.org/repos/asf/flink-statefun-playground.git

commit 54fb18dd5d5b16c74ed09603568755bbf0d2a7a0
Author: Till Rohrmann <tr...@apache.org>
AuthorDate: Sun Feb 6 13:14:30 2022 +0100

    [FLINK-25934] Reduce Statefun's memory consumption to 300mb
---
 playground-internal/statefun-playground-entrypoint/Dockerfile  |  4 +++-
 .../statefun-playground-entrypoint/docker-entrypoint.sh        | 10 ++++++++++
 .../internal/entrypoint/LocalEnvironmentEntrypoint.java        |  7 +++++++
 3 files changed, 20 insertions(+), 1 deletion(-)

diff --git a/playground-internal/statefun-playground-entrypoint/Dockerfile b/playground-internal/statefun-playground-entrypoint/Dockerfile
index 0438cc9..31bfb44 100644
--- a/playground-internal/statefun-playground-entrypoint/Dockerfile
+++ b/playground-internal/statefun-playground-entrypoint/Dockerfile
@@ -23,4 +23,6 @@ RUN mvn -f /usr/src/app/pom.xml clean package
 FROM openjdk:11
 WORKDIR /
 COPY --from=builder /usr/src/app/target/statefun-playground-entrypoint*.jar statefun-playground-entrypoint.jar
-ENTRYPOINT ["java", "-jar", "statefun-playground-entrypoint.jar"]
+COPY docker-entrypoint.sh /
+RUN chmod +x /docker-entrypoint.sh
+ENTRYPOINT ["/docker-entrypoint.sh"]
diff --git a/playground-internal/statefun-playground-entrypoint/docker-entrypoint.sh b/playground-internal/statefun-playground-entrypoint/docker-entrypoint.sh
new file mode 100644
index 0000000..f542d2f
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/docker-entrypoint.sh
@@ -0,0 +1,10 @@
+#!/usr/bin/env bash
+
+java \
+-XX:MaxDirectMemorySize=${MAX_DIRECT_MEMORY_SIZE:-80m} \
+-XX:MaxMetaspaceSize=${MAX_METASPACE_SIZE:-64m} \
+-Xms${MIN_HEAP_SIZE:-64m} \
+-Xmx${MAX_HEAP_SIZE:-192m} \
+-jar \
+statefun-playground-entrypoint.jar \
+"$@"
diff --git a/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java
index fb43e85..8503cf2 100644
--- a/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java
+++ b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java
@@ -21,7 +21,9 @@ import java.util.Collection;
 import org.apache.flink.api.java.utils.MultipleParameterTool;
 import org.apache.flink.configuration.CheckpointingOptions;
 import org.apache.flink.configuration.Configuration;
+import org.apache.flink.configuration.MemorySize;
 import org.apache.flink.configuration.StateBackendOptions;
+import org.apache.flink.configuration.TaskManagerOptions;
 import org.apache.flink.statefun.flink.core.StatefulFunctionsConfig;
 import org.apache.flink.statefun.flink.core.StatefulFunctionsJob;
 import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
@@ -96,6 +98,11 @@ public final class LocalEnvironmentEntrypoint {
     flinkConfiguration.set(StateBackendOptions.STATE_BACKEND, "rocksdb");
     flinkConfiguration.set(CheckpointingOptions.INCREMENTAL_CHECKPOINTS, true);
 
+    // reduce Flink's memory footprint a bit
+    flinkConfiguration.set(TaskManagerOptions.MANAGED_MEMORY_SIZE, MemorySize.ofMebiBytes(64));
+    flinkConfiguration.set(TaskManagerOptions.NETWORK_MEMORY_MIN, MemorySize.ofMebiBytes(16));
+    flinkConfiguration.set(TaskManagerOptions.NETWORK_MEMORY_MAX, MemorySize.ofMebiBytes(16));
+
     return flinkConfiguration;
   }
 }

[flink-statefun-playground] 01/04: [FLINK-25935] Add LocalEnvironmentEntrypoint

Posted by tr...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

trohrmann pushed a commit to branch release-3.2
in repository https://gitbox.apache.org/repos/asf/flink-statefun-playground.git

commit e956104c2e097891e03f62092844c687d984ebe0
Author: Till Rohrmann <tr...@apache.org>
AuthorDate: Fri Feb 4 19:07:22 2022 +0100

    [FLINK-25935] Add LocalEnvironmentEntrypoint
    
    The LocalEnvironmentEntrypoint runs a Stateful Functions applications within a
    single process. It uses Flink's LocalStreamExecutionEnvironment for this. The
    Entrypoint can be configured via
    
    * --module <PATH_TO_MODULE>
    * --set <CONFIG_OPTION>=<CONFIG_VALUE>
---
 .../statefun-playground-entrypoint/Dockerfile      |  26 ++++
 .../statefun-playground-entrypoint/README.md       |  12 ++
 .../statefun-playground-entrypoint/pom.xml         | 145 +++++++++++++++++++++
 .../entrypoint/ClassPathUniverseProvider.java      |  40 ++++++
 .../entrypoint/ConfigurationValidator.java         |  85 ++++++++++++
 .../entrypoint/LocalEnvironmentEntrypoint.java     | 101 ++++++++++++++
 .../src/main/resources/log4j.properties            |  20 +++
 7 files changed, 429 insertions(+)

diff --git a/playground-internal/statefun-playground-entrypoint/Dockerfile b/playground-internal/statefun-playground-entrypoint/Dockerfile
new file mode 100644
index 0000000..0438cc9
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/Dockerfile
@@ -0,0 +1,26 @@
+# Licensed to the Apache Software Foundation (ASF) under one or more
+# contributor license agreements.  See the NOTICE file distributed with
+# this work for additional information regarding copyright ownership.
+# The ASF licenses this file to You under the Apache License, Version 2.0
+# (the "License"); you may not use this file except in compliance with
+# the License.  You may obtain a copy of the License at
+#
+#     http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing, software
+# distributed under the License is distributed on an "AS IS" BASIS,
+# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+# See the License for the specific language governing permissions and
+# limitations under the License.
+
+# Build the functions code ...
+FROM maven:3.6.3-jdk-11 AS builder
+COPY src /usr/src/app/src
+COPY pom.xml /usr/src/app
+RUN mvn -f /usr/src/app/pom.xml clean package
+
+# ... and run the web server!
+FROM openjdk:11
+WORKDIR /
+COPY --from=builder /usr/src/app/target/statefun-playground-entrypoint*.jar statefun-playground-entrypoint.jar
+ENTRYPOINT ["java", "-jar", "statefun-playground-entrypoint.jar"]
diff --git a/playground-internal/statefun-playground-entrypoint/README.md b/playground-internal/statefun-playground-entrypoint/README.md
new file mode 100644
index 0000000..07d1302
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/README.md
@@ -0,0 +1,12 @@
+# Stateful Functions Playground Entrypoint
+
+A simple Stateful Functions entrypoint that runs Stateful Functions within a single process.
+
+## Configuring a module.yaml
+
+Per default the `LocalEnvironmentEntrypoint` expects a `module.yaml` to be on the classpath.
+Alternatively, one can provide a different location via `--module file://<PATH>`.
+
+## Configuring the Flink Runtime
+
+One can configure the underlying Flink runtime via `--set <CONFIG_OPTION>=<CONFIG_VALUE>`.
\ No newline at end of file
diff --git a/playground-internal/statefun-playground-entrypoint/pom.xml b/playground-internal/statefun-playground-entrypoint/pom.xml
new file mode 100644
index 0000000..77cd8ca
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/pom.xml
@@ -0,0 +1,145 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+Licensed to the Apache Software Foundation (ASF) under one
+or more contributor license agreements.  See the NOTICE file
+distributed with this work for additional information
+regarding copyright ownership.  The ASF licenses this file
+to you under the Apache License, Version 2.0 (the
+"License"); you may not use this file except in compliance
+with the License.  You may obtain a copy of the License at
+  http://www.apache.org/licenses/LICENSE-2.0
+Unless required by applicable law or agreed to in writing,
+software distributed under the License is distributed on an
+"AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+KIND, either express or implied.  See the License for the
+specific language governing permissions and limitations
+under the License.
+-->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+         xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+         xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
+    <modelVersion>4.0.0</modelVersion>
+
+    <groupId>org.apache.flink</groupId>
+    <artifactId>statefun-playground-entrypoint</artifactId>
+    <version>3.2.0</version>
+    <packaging>jar</packaging>
+
+    <properties>
+        <statefun.version>3.2.0</statefun.version>
+        <flink.version>1.14.3</flink.version>
+        <slf4j.version>1.7.35</slf4j.version>
+        <maven.compiler.source>11</maven.compiler.source>
+        <maven.compiler.target>11</maven.compiler.target>
+    </properties>
+
+    <dependencies>
+        <!-- StateFun Core -->
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>statefun-flink-core</artifactId>
+            <version>${statefun.version}</version>
+        </dependency>
+        <!-- StateFun Distribution -->
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>statefun-flink-io-bundle</artifactId>
+            <version>${statefun.version}</version>
+        </dependency>
+
+        <!-- Flink dependencies -->
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>flink-java</artifactId>
+            <version>${flink.version}</version>
+        </dependency>
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>flink-streaming-java_2.12</artifactId>
+            <version>${flink.version}</version>
+        </dependency>
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>flink-clients_2.12</artifactId>
+            <version>${flink.version}</version>
+        </dependency>
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>flink-runtime-web_2.12</artifactId>
+            <version>${flink.version}</version>
+        </dependency>
+        <dependency>
+            <groupId>org.apache.flink</groupId>
+            <artifactId>flink-statebackend-rocksdb_2.12</artifactId>
+            <version>${flink.version}</version>
+        </dependency>
+
+        <!-- Logging -->
+        <dependency>
+            <groupId>org.slf4j</groupId>
+            <artifactId>slf4j-api</artifactId>
+            <version>${slf4j.version}</version>
+        </dependency>
+        <dependency>
+            <groupId>org.slf4j</groupId>
+            <artifactId>slf4j-log4j12</artifactId>
+            <version>${slf4j.version}</version>
+        </dependency>
+    </dependencies>
+
+    <build>
+        <plugins>
+            <!-- Build a fat executable jar -->
+            <plugin>
+                <groupId>org.apache.maven.plugins</groupId>
+                <artifactId>maven-shade-plugin</artifactId>
+                <version>3.2.4</version>
+                <executions>
+                    <execution>
+                        <phase>package</phase>
+                        <goals>
+                            <goal>shade</goal>
+                        </goals>
+                        <configuration>
+                            <createDependencyReducedPom>false</createDependencyReducedPom>
+                            <transformers>
+                                <transformer implementation="org.apache.maven.plugins.shade.resource.ManifestResourceTransformer">
+                                    <mainClass>org.apache.flink.statefun.playground.internal.entrypoint.LocalEnvironmentEntrypoint</mainClass>
+                                </transformer>
+                                <transformer
+                                        implementation="org.apache.maven.plugins.shade.resource.ServicesResourceTransformer"/>
+                                <transformer
+                                        implementation="org.apache.maven.plugins.shade.resource.ApacheLicenseResourceTransformer"/>
+                            </transformers>
+                        </configuration>
+                    </execution>
+                </executions>
+            </plugin>
+
+            <!-- Java code style -->
+            <plugin>
+                <groupId>com.diffplug.spotless</groupId>
+                <artifactId>spotless-maven-plugin</artifactId>
+                <version>1.20.0</version>
+                <configuration>
+                    <java>
+                        <googleJavaFormat>
+                            <version>1.7</version>
+                            <style>GOOGLE</style>
+                        </googleJavaFormat>
+                        <removeUnusedImports/>
+                    </java>
+                </configuration>
+                <executions>
+                    <execution>
+                        <id>spotless-check</id>
+                        <phase>verify</phase>
+                        <goals>
+                            <goal>check</goal>
+                        </goals>
+                    </execution>
+                </executions>
+            </plugin>
+        </plugins>
+    </build>
+</project>
diff --git a/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ClassPathUniverseProvider.java b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ClassPathUniverseProvider.java
new file mode 100644
index 0000000..dfd48a9
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ClassPathUniverseProvider.java
@@ -0,0 +1,40 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.flink.statefun.playground.internal.entrypoint;
+
+import org.apache.flink.statefun.flink.core.StatefulFunctionsConfig;
+import org.apache.flink.statefun.flink.core.StatefulFunctionsUniverse;
+import org.apache.flink.statefun.flink.core.StatefulFunctionsUniverseProvider;
+import org.apache.flink.statefun.flink.core.spi.Modules;
+
+/**
+ * This class duplicates StatefulFunctionsUniverses.ClassPathUniverseProvider because it is not
+ * public. Needs a new Statefun release to change the visibility. @Todo Update Statefun to expose
+ * this functionality
+ */
+final class ClassPathUniverseProvider implements StatefulFunctionsUniverseProvider {
+
+  private static final long serialVersionUID = 1;
+
+  @Override
+  public StatefulFunctionsUniverse get(
+      ClassLoader classLoader, StatefulFunctionsConfig configuration) {
+    Modules modules = Modules.loadFromClassPath(configuration);
+    return modules.createStatefulFunctionsUniverse();
+  }
+}
diff --git a/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ConfigurationValidator.java b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ConfigurationValidator.java
new file mode 100644
index 0000000..133c8cb
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/ConfigurationValidator.java
@@ -0,0 +1,85 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.flink.statefun.playground.internal.entrypoint;
+
+import org.apache.flink.configuration.ConfigOption;
+import org.apache.flink.configuration.ConfigOptions;
+import org.apache.flink.configuration.Configuration;
+import org.apache.flink.statefun.flink.core.StatefulFunctionsConfig;
+import org.apache.flink.statefun.flink.core.exceptions.StatefulFunctionsInvalidConfigException;
+import org.apache.flink.statefun.flink.core.message.MessageFactoryType;
+import org.apache.flink.util.StringUtils;
+
+public final class ConfigurationValidator {
+
+  private ConfigurationValidator() {}
+
+  static void validate(Configuration configuration) {
+    validateCustomPayloadSerializerClassName(configuration);
+    validateNoHeapBackedTimers(configuration);
+    validateUnalignedCheckpointsDisabled(configuration);
+  }
+
+  private static void validateCustomPayloadSerializerClassName(Configuration configuration) {
+
+    MessageFactoryType factoryType =
+        configuration.get(StatefulFunctionsConfig.USER_MESSAGE_SERIALIZER);
+    String customPayloadSerializerClassName =
+        configuration.get(StatefulFunctionsConfig.USER_MESSAGE_CUSTOM_PAYLOAD_SERIALIZER_CLASS);
+
+    if (factoryType == MessageFactoryType.WITH_CUSTOM_PAYLOADS) {
+      if (StringUtils.isNullOrWhitespaceOnly(customPayloadSerializerClassName)) {
+        throw new StatefulFunctionsInvalidConfigException(
+            StatefulFunctionsConfig.USER_MESSAGE_CUSTOM_PAYLOAD_SERIALIZER_CLASS,
+            "custom payload serializer class must be supplied with WITH_CUSTOM_PAYLOADS serializer");
+      }
+    } else {
+      if (customPayloadSerializerClassName != null) {
+        throw new StatefulFunctionsInvalidConfigException(
+            StatefulFunctionsConfig.USER_MESSAGE_CUSTOM_PAYLOAD_SERIALIZER_CLASS,
+            "custom payload serializer class may only be supplied with WITH_CUSTOM_PAYLOADS serializer");
+      }
+    }
+  }
+
+  private static final ConfigOption<String> TIMER_SERVICE_FACTORY =
+      ConfigOptions.key("state.backend.rocksdb.timer-service.factory")
+          .stringType()
+          .defaultValue("rocksdb");
+
+  private static final ConfigOption<Boolean> ENABLE_UNALIGNED_CHECKPOINTS =
+      ConfigOptions.key("execution.checkpointing.unaligned").booleanType().defaultValue(false);
+
+  private static void validateNoHeapBackedTimers(Configuration configuration) {
+    final String timerFactory = configuration.getString(TIMER_SERVICE_FACTORY);
+    if (!timerFactory.equalsIgnoreCase("rocksdb")) {
+      throw new StatefulFunctionsInvalidConfigException(
+          TIMER_SERVICE_FACTORY,
+          "StateFun only supports non-heap timers with a rocksdb state backend.");
+    }
+  }
+
+  private static void validateUnalignedCheckpointsDisabled(Configuration configuration) {
+    final boolean unalignedCheckpoints = configuration.getBoolean(ENABLE_UNALIGNED_CHECKPOINTS);
+    if (unalignedCheckpoints) {
+      throw new StatefulFunctionsInvalidConfigException(
+          ENABLE_UNALIGNED_CHECKPOINTS,
+          "StateFun currently does not support unaligned checkpointing.");
+    }
+  }
+}
diff --git a/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java
new file mode 100644
index 0000000..fb43e85
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/src/main/java/org/apache/flink/statefun/playground/internal/entrypoint/LocalEnvironmentEntrypoint.java
@@ -0,0 +1,101 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.flink.statefun.playground.internal.entrypoint;
+
+import java.util.Collection;
+import org.apache.flink.api.java.utils.MultipleParameterTool;
+import org.apache.flink.configuration.CheckpointingOptions;
+import org.apache.flink.configuration.Configuration;
+import org.apache.flink.configuration.StateBackendOptions;
+import org.apache.flink.statefun.flink.core.StatefulFunctionsConfig;
+import org.apache.flink.statefun.flink.core.StatefulFunctionsJob;
+import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Entrypoint that starts a local Flink environment to run the given Stateful Functions application
+ * in this process.
+ */
+public final class LocalEnvironmentEntrypoint {
+  private static final Logger LOG = LoggerFactory.getLogger(LocalEnvironmentEntrypoint.class);
+
+  private static final String MODULE_OPTION = "module";
+  private static final String CONFIGURATION_OPTION = "set";
+
+  public static void main(String[] args) throws Exception {
+    final Configuration flinkConfiguration = parseConfiguration(args);
+    ConfigurationValidator.validate(flinkConfiguration);
+
+    final StreamExecutionEnvironment env =
+        StreamExecutionEnvironment.createLocalEnvironmentWithWebUI(flinkConfiguration);
+
+    final StatefulFunctionsConfig stateFunConfig =
+        StatefulFunctionsConfig.fromFlinkConfiguration(flinkConfiguration);
+    stateFunConfig.setProvider(new ClassPathUniverseProvider());
+
+    StatefulFunctionsJob.main(env, stateFunConfig);
+  }
+
+  private static Configuration parseConfiguration(String[] args) {
+    final MultipleParameterTool parameterTool = MultipleParameterTool.fromArgs(args);
+    final Configuration flinkConfiguration = createDefaultLocalEnvironmentFlinkConfiguration();
+    parseModuleOption(parameterTool, flinkConfiguration);
+    parseConfigurationOptions(parameterTool, flinkConfiguration);
+
+    return flinkConfiguration;
+  }
+
+  private static void parseConfigurationOptions(
+      MultipleParameterTool parameterTool, Configuration flinkConfiguration) {
+    final Collection<String> configurationOptions =
+        parameterTool.getMultiParameter(CONFIGURATION_OPTION);
+
+    if (configurationOptions != null) {
+      for (String configurationOption : configurationOptions) {
+        final String[] splits = configurationOption.split("=");
+
+        if (splits.length != 2) {
+          throw new IllegalArgumentException(
+              String.format("The '--%s' value must have the form 'key=value'", CONFIGURATION_OPTION));
+        }
+
+        final String key = splits[0];
+        final String value = splits[1];
+        LOG.info("Setting configuration value: {}={}", key, value);
+        flinkConfiguration.setString(key, value);
+      }
+    }
+  }
+
+  private static void parseModuleOption(
+      MultipleParameterTool parameterTool, Configuration flinkConfiguration) {
+    final String module = parameterTool.get(MODULE_OPTION, "file:///module.yaml");
+
+    LOG.info("Setting module.yaml to: {}", module);
+    flinkConfiguration.set(StatefulFunctionsConfig.REMOTE_MODULE_NAME, module);
+  }
+
+  private static Configuration createDefaultLocalEnvironmentFlinkConfiguration() {
+    final Configuration flinkConfiguration = new Configuration();
+    flinkConfiguration.set(StateBackendOptions.STATE_BACKEND, "rocksdb");
+    flinkConfiguration.set(CheckpointingOptions.INCREMENTAL_CHECKPOINTS, true);
+
+    return flinkConfiguration;
+  }
+}
diff --git a/playground-internal/statefun-playground-entrypoint/src/main/resources/log4j.properties b/playground-internal/statefun-playground-entrypoint/src/main/resources/log4j.properties
new file mode 100644
index 0000000..8d9b95d
--- /dev/null
+++ b/playground-internal/statefun-playground-entrypoint/src/main/resources/log4j.properties
@@ -0,0 +1,20 @@
+# Licensed to the Apache Software Foundation (ASF) under one or more
+# contributor license agreements.  See the NOTICE file distributed with
+# this work for additional information regarding copyright ownership.
+# The ASF licenses this file to You under the Apache License, Version 2.0
+# (the "License"); you may not use this file except in compliance with
+# the License.  You may obtain a copy of the License at
+#
+#     http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing, software
+# distributed under the License is distributed on an "AS IS" BASIS,
+# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+# See the License for the specific language governing permissions and
+# limitations under the License.
+
+log4j.rootLogger=INFO, console
+
+log4j.appender.console=org.apache.log4j.ConsoleAppender
+log4j.appender.console.layout=org.apache.log4j.PatternLayout
+log4j.appender.console.layout.ConversionPattern=%d{HH:mm:ss,SSS} %-5p %-60c %x - %m%n

[flink-statefun-playground] 03/04: [FLINK-25934] Replace Kafka image with Redpanda

Posted by tr...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

trohrmann pushed a commit to branch release-3.2
in repository https://gitbox.apache.org/repos/asf/flink-statefun-playground.git

commit d1fa6dd069979831b13ae933a1ee7ff651363133
Author: Till Rohrmann <tr...@apache.org>
AuthorDate: Sun Feb 6 12:11:43 2022 +0100

    [FLINK-25934] Replace Kafka image with Redpanda
---
 go/greeter/README.md                         | 14 +++++-------
 go/greeter/docker-compose.yml                | 30 +++++++++++++-------------
 go/showcase/docker-compose.yml               | 30 +++++++++++++-------------
 java/connected-components/README.md          |  5 +----
 java/connected-components/docker-compose.yml | 30 +++++++++++++-------------
 java/greeter/README.md                       |  5 +----
 java/greeter/docker-compose.yml              | 32 +++++++++++++---------------
 java/shopping-cart/README.md                 |  2 +-
 java/shopping-cart/docker-compose.yml        | 30 +++++++++++++-------------
 javascript/greeter/README.md                 | 14 +++++-------
 javascript/greeter/docker-compose.yml        | 30 +++++++++++++-------------
 javascript/showcase/docker-compose.yml       | 30 +++++++++++++-------------
 python/greeter/README.md                     | 14 +++++-------
 python/greeter/docker-compose.yml            | 30 +++++++++++++-------------
 python/showcase/docker-compose.yml           | 30 +++++++++++++-------------
 15 files changed, 153 insertions(+), 173 deletions(-)

diff --git a/go/greeter/README.md b/go/greeter/README.md
index 44b18c6..5241620 100644
--- a/go/greeter/README.md
+++ b/go/greeter/README.md
@@ -2,11 +2,11 @@
 
 This is a simple example of a stateful functions application implemented in `Go`.
 
-In this example, we imagine a service that computes personalized greetings. 
+In this example, we imagine a service that computes personalized greetings.
 Our service, consist out of the following components:
 
 * `kafka ingress` - This component forwards messages produced to the `names` kafka topic,
-to the `person` stateful function. Messages produced to this topic has the following 
+to the `person` stateful function. Messages produced to this topic has the following
 schema `{ "name" : "bob"}`.
 
 * `person` - This function is triggered by the ingress defined above.
@@ -14,7 +14,7 @@ This function keeps track of the number of visits, and triggers the next functio
 
 * `greeter` - This function, computes a personalized greeting, based on the name and the number
 of visits of that user. The output of that computation is forward to a Kafka egress defined below.
- 
+
 * `kafka egress` - This wraps a Kafka producer that emits `utf-8` greetings to the `greetings` Kafka topic.
 
 
@@ -30,11 +30,7 @@ docker-compose up
 To observe the customized greeting, as they appear in the `greetings` Kafka topic, run in a separate terminal:
 
 ```
-docker-compose exec kafka kafka-console-consumer \
-     --bootstrap-server kafka:9092 \
-     --isolation-level read_committed \
-     --from-beginning \
-     --topic greetings
+docker-compose exec kafka rpk topic consume greetings
 ```
 
 Try adding few more input lines to [input-example.json](input-example.json), and restart
@@ -42,7 +38,7 @@ the producer service.
 
 ```
 docker-compose restart producer
-``` 
+```
 
 Feeling curious? add the following print to the `person` function at [greeter.go](greeter.go):
 ```fmt.Printf("Hello there %d!", ctx.Self().Id)```.
diff --git a/go/greeter/docker-compose.yml b/go/greeter/docker-compose.yml
index f72a966..8e4e8a3 100644
--- a/go/greeter/docker-compose.yml
+++ b/go/greeter/docker-compose.yml
@@ -47,24 +47,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   #    Simple Kafka JSON producer to simulate ingress events
diff --git a/go/showcase/docker-compose.yml b/go/showcase/docker-compose.yml
index f91c0a1..c493c19 100644
--- a/go/showcase/docker-compose.yml
+++ b/go/showcase/docker-compose.yml
@@ -36,24 +36,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   # Forward a port 8000 from the host's machine
diff --git a/java/connected-components/README.md b/java/connected-components/README.md
index 981a754..17cfa82 100644
--- a/java/connected-components/README.md
+++ b/java/connected-components/README.md
@@ -54,10 +54,7 @@ $ docker-compose up
 You can take a look at what messages are being sent to the Kafka egress:
 
 ```
-$ docker-compose exec kafka kafka-console-consumer \
-      --bootstrap-server kafka:9092 \
-      --topic connected-component-changes \
-      --from-beginning
+$ docker-compose exec kafka rpk topic consume connected-component-changes
 ```
 
 You can also try modifying the function code in the `src/main/java` directory, and do a zero-downtime upgrade of the
diff --git a/java/connected-components/docker-compose.yml b/java/connected-components/docker-compose.yml
index d66fc1f..3bbb2e6 100644
--- a/java/connected-components/docker-compose.yml
+++ b/java/connected-components/docker-compose.yml
@@ -48,24 +48,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   #    Simple Kafka JSON producer to simulate ingress events
diff --git a/java/greeter/README.md b/java/greeter/README.md
index 315b278..739b5f5 100644
--- a/java/greeter/README.md
+++ b/java/greeter/README.md
@@ -54,10 +54,7 @@ $ docker-compose up
 You can take a look at what messages are being sent to the Kafka egress:
 
 ```
-$ docker-compose exec kafka kafka-console-consumer \
-      --bootstrap-server kafka:9092 \
-      --topic greetings \
-      --from-beginning
+$ docker-compose exec kafka rpk topic consume greetings
 ```
 
 You can also try modifying the function code in the `src/main/java` directory, and do a zero-downtime upgrade of the
diff --git a/java/greeter/docker-compose.yml b/java/greeter/docker-compose.yml
index fec846d..cc2e1b1 100644
--- a/java/greeter/docker-compose.yml
+++ b/java/greeter/docker-compose.yml
@@ -48,26 +48,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    links:
-      - "zookeeper:zookeeper"
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   #    Simple Kafka JSON producer to simulate ingress events
diff --git a/java/shopping-cart/README.md b/java/shopping-cart/README.md
index 1d8b539..c1877ed 100644
--- a/java/shopping-cart/README.md
+++ b/java/shopping-cart/README.md
@@ -63,7 +63,7 @@ Note: `Caller: Optional.empty` in the logs corresponds to the messages that came
 
 To see the results produced to the egress:
 ```
-docker-compose exec kafka bash -c '/usr/bin/kafka-console-consumer --topic receipts --bootstrap-server kafka:9092'
+docker-compose exec kafka rpk topic consume receipts'
 ```
 
 If you want to modify the code, you can do a hot redeploy of your functions service:
diff --git a/java/shopping-cart/docker-compose.yml b/java/shopping-cart/docker-compose.yml
index 3a2d25e..e4be3aa 100644
--- a/java/shopping-cart/docker-compose.yml
+++ b/java/shopping-cart/docker-compose.yml
@@ -48,21 +48,21 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
diff --git a/javascript/greeter/README.md b/javascript/greeter/README.md
index efc2bd1..ea1fa0a 100644
--- a/javascript/greeter/README.md
+++ b/javascript/greeter/README.md
@@ -2,11 +2,11 @@
 
 This is a simple example of a stateful functions application implemented in `JavaScript`.
 
-In this example, we imagine a service that computes personalized greetings. 
+In this example, we imagine a service that computes personalized greetings.
 Our service, consist out of the following components:
 
 * `kafka ingress` - This component forwards messages produced to the `names` kafka topic,
-to the `person` stateful function. Messages produced to this topic has the following 
+to the `person` stateful function. Messages produced to this topic has the following
 schema `{ "name" : "bob"}`.
 
 * `person` - This function is triggered by the ingress defined above.
@@ -14,7 +14,7 @@ This function keeps track of the number of visits, and triggers the next functio
 
 * `greeter` - This function, computes a personalized greeting, based on the name and the number
 of visits of that user. The output of that computation is forward to a Kafka egress defined below.
- 
+
 * `kafka egress` - This wraps a Kafka producer that emits `utf-8` greetings to the `greetings` Kafka topic.
 
 
@@ -30,11 +30,7 @@ docker-compose up
 To observe the customized greeting, as they appear in the `greetings` Kafka topic, run in a separate terminal:
 
 ```
-docker-compose exec kafka kafka-console-consumer \
-     --bootstrap-server kafka:9092 \
-     --isolation-level read_committed \
-     --from-beginning \
-     --topic greetings
+docker-compose exec kafka rpk topic consume greetings
 ```
 
 Try adding few more input lines to [input-example.json](input-example.json), and restart
@@ -42,7 +38,7 @@ the producer service.
 
 ```
 docker-compose restart producer
-``` 
+```
 
 Feeling curious? add the following log to the `person` function at [functions.js](functions.js):
 ```console.log(`Hello there ${context.self.id}!`);```.
diff --git a/javascript/greeter/docker-compose.yml b/javascript/greeter/docker-compose.yml
index f72a966..8e4e8a3 100644
--- a/javascript/greeter/docker-compose.yml
+++ b/javascript/greeter/docker-compose.yml
@@ -47,24 +47,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   #    Simple Kafka JSON producer to simulate ingress events
diff --git a/javascript/showcase/docker-compose.yml b/javascript/showcase/docker-compose.yml
index e07f6d6..2494a6f 100644
--- a/javascript/showcase/docker-compose.yml
+++ b/javascript/showcase/docker-compose.yml
@@ -36,24 +36,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   # Forward a port 8000 from the host's machine
diff --git a/python/greeter/README.md b/python/greeter/README.md
index 72996b9..0a5aad8 100644
--- a/python/greeter/README.md
+++ b/python/greeter/README.md
@@ -2,11 +2,11 @@
 
 This is a simple example of a stateful functions application implemented in `Python`.
 
-In this example, we imagine a service that computes personalized greetings. 
+In this example, we imagine a service that computes personalized greetings.
 Our service, consist out of the following components:
 
 * `kafka ingress` - This component forwards messages produced to the `names` kafka topic,
-to the `person` stateful function. Messages produced to this topic has the following 
+to the `person` stateful function. Messages produced to this topic has the following
 schema `{ "name" : "bob"}`.
 
 * `person` - This function is triggered by the ingress defined above.
@@ -14,7 +14,7 @@ This function keeps track of the number of visits, and triggers the next functio
 
 * `greeter` - This function, computes a personalized greeting, based on the name and the number
 of visits of that user. The output of that computation is forward to a Kafka egress defined below.
- 
+
 * `kafka egress` - This wraps a Kafka producer that emits `utf-8` greetings to the `greetings` Kafka topic.
 
 
@@ -30,11 +30,7 @@ docker-compose up
 To observe the customized greeting, as they appear in the `greetings` Kafka topic, run in a separate terminal:
 
 ```
-docker-compose exec kafka kafka-console-consumer \
-     --bootstrap-server kafka:9092 \
-     --isolation-level read_committed \
-     --from-beginning \
-     --topic greetings
+docker-compose exec kafka rpk topic consume greetings
 ```
 
 Try adding few more input lines to [input-example.json](input-example.json), and restart
@@ -42,7 +38,7 @@ the producer service.
 
 ```
 docker-compose restart producer
-``` 
+```
 
 Feeling curious? add the following print to the `person` function at [functions.py](functions.py):
 ```print(f"Hello there {context.address.id}!", flush=True)```.
diff --git a/python/greeter/docker-compose.yml b/python/greeter/docker-compose.yml
index f72a966..8e4e8a3 100644
--- a/python/greeter/docker-compose.yml
+++ b/python/greeter/docker-compose.yml
@@ -47,24 +47,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   #    Simple Kafka JSON producer to simulate ingress events
diff --git a/python/showcase/docker-compose.yml b/python/showcase/docker-compose.yml
index e07f6d6..2494a6f 100644
--- a/python/showcase/docker-compose.yml
+++ b/python/showcase/docker-compose.yml
@@ -36,24 +36,24 @@ services:
   #    Kafka for ingress and egress
   ###############################################################
 
-  zookeeper:
-    image: confluentinc/cp-zookeeper:5.4.3
-    environment:
-      ZOOKEEPER_CLIENT_PORT: "2181"
-    ports:
-      - "2181:2181"
-
   kafka:
-    image: confluentinc/cp-kafka:5.4.3
+    image: docker.vectorized.io/vectorized/redpanda:v21.8.1
+    command:
+      - redpanda start
+      - --smp 1
+      - --memory 512M
+      - --overprovisioned
+      - --set redpanda.default_topic_replications=1
+      - --set redpanda.auto_create_topics_enabled=true
+      - --kafka-addr INSIDE://0.0.0.0:9094,OUTSIDE://0.0.0.0:9092
+      - --advertise-kafka-addr INSIDE://kafka:9094,OUTSIDE://kafka:9092
+      - --pandaproxy-addr 0.0.0.0:8089
+      - --advertise-pandaproxy-addr kafka:8089
+    hostname: kafka
     ports:
+      - "8089:8089"
       - "9092:9092"
-    depends_on:
-      - zookeeper
-    environment:
-      KAFKA_ZOOKEEPER_CONNECT: zookeeper:2181
-      KAFKA_ADVERTISED_LISTENERS: PLAINTEXT://kafka:9092
-      KAFKA_AUTO_CREATE_TOPICS_ENABLE: "true"
-      KAFKA_OFFSETS_TOPIC_REPLICATION_FACTOR: 1
+      - "9094:9094"
 
   ###############################################################
   # Forward a port 8000 from the host's machine