Skip to content
Merged
Show file tree
Hide file tree
Changes from 1 commit
Commits
Show all changes
28 commits
Select commit Hold shift + click to select a range
328c44a
add kafka security module
ekawinataa Sep 2, 2024
1f6d09c
aDD SASL class callback config for producer and consumer
ekawinataa Sep 2, 2024
19574f9
Add config map
ekawinataa Sep 2, 2024
28e31de
remove build.gradle
ekawinataa Sep 2, 2024
6c33ff4
Add dynamic props
ekawinataa Sep 3, 2024
8ef873b
Update regex
ekawinataa Sep 3, 2024
c3d4556
rename var
ekawinataa Sep 3, 2024
f19b90b
Remove redundant imports
ekawinataa Sep 3, 2024
b731a38
Rename prefix
ekawinataa Sep 3, 2024
ef3d7e9
Remove unused import
ekawinataa Sep 3, 2024
e643dc6
Update test
ekawinataa Sep 3, 2024
d9cfc8a
Add implementation for sink dynamic props
ekawinataa Sep 5, 2024
fc26377
Add null checking for the additional props
ekawinataa Sep 5, 2024
d2c981d
Added validations on source config
ekawinataa Sep 10, 2024
11c8da5
Add docs and refactor pattern to enum
ekawinataa Sep 10, 2024
fb8695b
chECKSTYLE
ekawinataa Sep 10, 2024
cc2485f
Add readme
ekawinataa Sep 10, 2024
d110153
Make the pattern more specific and embedded to the enum
ekawinataa Sep 11, 2024
33b0cf9
Add more test
ekawinataa Sep 11, 2024
5f3a80c
bump version
ekawinataa Sep 11, 2024
62b4eca
Add unit tests
ekawinataa Sep 13, 2024
6c4e8ae
Use expected annotation
ekawinataa Sep 16, 2024
1063b25
Assert exception message. Add fail mechanism in case of not throwing …
ekawinataa Sep 16, 2024
07f5e53
Use rule for asserting exception
ekawinataa Sep 16, 2024
edf2e4d
Add more test case
ekawinataa Sep 16, 2024
0c2d621
add more unit test
ekawinataa Sep 16, 2024
a262868
feat: Enable multiple underscore parsing
ekawinataa Sep 16, 2024
d31f834
test: Add test on multiple underscore parsing
ekawinataa Sep 16, 2024
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Prev Previous commit
Next Next commit
Add docs and refactor pattern to enum
  • Loading branch information
ekawinataa committed Sep 10, 2024
commit 11c8da56ec89f8c6e54ef5da5ddf3c39ac8adbce
Original file line number Diff line number Diff line change
@@ -1,5 +1,14 @@
package com.gotocompany.dagger.core.enumeration;

import lombok.Getter;

@Getter
public enum ConnectorType {
SINK, SOURCE
SOURCE("SOURCE_KAFKA_CONSUMER_CONFIG_(.*)"), SINK("SINK_KAFKA_PRODUCER_CONFIG_(.*)");

ConnectorType(String prefixPattern) {
this.prefixPattern = prefixPattern;
}

private final String prefixPattern;
}
Original file line number Diff line number Diff line change
Expand Up @@ -15,8 +15,8 @@ public class KafkaConfigUtil {

static {
CONFIG_PATTERN = new HashMap<>();
CONFIG_PATTERN.put(ConnectorType.SOURCE, Pattern.compile("SOURCE_KAFKA_CONSUMER_CONFIG_(.*)", Pattern.CASE_INSENSITIVE));
CONFIG_PATTERN.put(ConnectorType.SINK, Pattern.compile("SINK_KAFKA_PRODUCER_CONFIG_(.*)", Pattern.CASE_INSENSITIVE));
CONFIG_PATTERN.put(ConnectorType.SOURCE, Pattern.compile(ConnectorType.SOURCE.getPrefixPattern(), Pattern.CASE_INSENSITIVE));
CONFIG_PATTERN.put(ConnectorType.SINK, Pattern.compile(ConnectorType.SINK.getPrefixPattern(), Pattern.CASE_INSENSITIVE));
}

public static Properties parseKafkaConfiguration(ConnectorType connectorType, Properties properties) {
Expand Down
37 changes: 37 additions & 0 deletions docs/docs/guides/kafka.md
Original file line number Diff line number Diff line change
@@ -0,0 +1,37 @@
# Kafka

Kafka topics are used as the source and output of daggers. Both of source and output kafka configurations are defined through the properties file.

## Source Kafka Configuration

There can be multiple source kafka configurations in the properties file. Source configurations are defined through `STREAMS` property.
Here are the predefined properties for source kafka configuration:

- SOURCE_KAFKA_CONSUMER_CONFIG_AUTO_COMMIT_ENABLE
- SOURCE_KAFKA_CONSUMER_CONFIG_GROUP_ID
- SOURCE_KAFKA_CONSUMER_CONFIG_BOOTSTRAP_SERVERS
- SOURCE_KAFKA_CONSUMER_CONFIG_SECURITY_PROTOCOL
- SOURCE_KAFKA_CONSUMER_CONFIG_SASL_MECHANISM
- SOURCE_KAFKA_CONSUMER_CONFIG_SASL_JAAS_CONFIG
- SOURCE_KAFKA_CONSUMER_ADDITIONAL_CONFIGURATIONS
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_KEY_PASSWORD
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_KEYSTORE_LOCATION
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_KEYSTORE_PASSWORD
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_KEYSTORE_TYPE
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_TRUSTSTORE_LOCATION
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_TRUSTSTORE_PASSWORD
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_TRUSTSTORE_TYPE
- SOURCE_KAFKA_CONSUMER_CONFIG_SSL_PROTOCOL

Additional kafka configuration can be passed through the `SOURCE_KAFKA_CONSUMER_ADDITIONAL_CONFIGURATIONS` property. This property should be a json key-value map.
For example :
- SOURCE_KAFKA_CONSUMER_ADDITIONAL_CONFIGURATIONS={"SOURCE_KAFKA_CONSUMER_CONFIG_KEY_DESERIALIZER":"org.apache.kafka.common.serialization.StringDeserializer","SOURCE_KAFKA_CONSUMER_CONFIG_VALUE_DESERIALIZER":"org.apache.kafka.common.serialization.StringDeserializer"}


## Sink Kafka Configuration

There is only one sink kafka configuration in the properties file. Sink configuration is defined by properties having `SINK_KAFKA_PRODUCER_CONFIG`

```properties

```bash