-
Notifications
You must be signed in to change notification settings - Fork 101
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
[SNOW-954150] Use map of clients with different configurations instead of one client for multiple connectors configurations #744
Changes from 49 commits
1ca1d32
15da6b5
f5fe7d8
7426bc9
a316c0e
07659a2
d5e7e20
4c7a3a5
6cce84d
a14f388
e18292a
e762419
c2f9a6a
9fbd4e5
02a8c58
3b4e729
89109c1
d87cdf3
4f85c16
15953d5
0ce288a
7628b75
7014658
123d4c4
49c72d4
df0aaa7
275129c
fffaead
7d43de4
87fd612
97e9aa5
369c55d
5ef97aa
f2df6fb
2db159c
a076c51
1ffd3eb
8f0c73c
f75204d
af10de3
d6575ed
2db9438
5d7f683
314c658
4b4e25f
9e2a386
99e0359
381e1c8
410b0d0
377894c
b937485
94b3797
f7d6e5e
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change |
---|---|---|
|
@@ -17,15 +17,8 @@ | |
|
||
package com.snowflake.kafka.connector.internal.streaming; | ||
|
||
import static com.snowflake.kafka.connector.SnowflakeSinkConnectorConfig.SNOWPIPE_STREAMING_FILE_VERSION; | ||
import static net.snowflake.ingest.utils.ParameterProvider.BLOB_FORMAT_VERSION; | ||
|
||
import com.snowflake.kafka.connector.Utils; | ||
import com.snowflake.kafka.connector.internal.KCLogger; | ||
import java.util.HashMap; | ||
import java.util.Map; | ||
import java.util.Optional; | ||
import java.util.Properties; | ||
import java.util.concurrent.atomic.AtomicInteger; | ||
import net.snowflake.ingest.streaming.SnowflakeStreamingIngestClient; | ||
import net.snowflake.ingest.streaming.SnowflakeStreamingIngestClientFactory; | ||
|
@@ -35,9 +28,6 @@ | |
/** This class handles all calls to manage the streaming ingestion client */ | ||
public class StreamingClientHandler { | ||
private static final KCLogger LOGGER = new KCLogger(StreamingClientHandler.class.getName()); | ||
private static final String STREAMING_CLIENT_PREFIX_NAME = "KC_CLIENT_"; | ||
private static final String TEST_CLIENT_NAME = "TEST_CLIENT"; | ||
|
||
private AtomicInteger createdClientId = new AtomicInteger(0); | ||
|
||
/** | ||
|
@@ -51,40 +41,27 @@ public static boolean isClientValid(SnowflakeStreamingIngestClient client) { | |
} | ||
|
||
/** | ||
* Creates a streaming client from the given config | ||
* Creates a streaming client from the given properties | ||
* | ||
* @param connectorConfig The config to create the client | ||
* @param streamingClientProperties The properties to create the client | ||
* @return A newly created client | ||
*/ | ||
public SnowflakeStreamingIngestClient createClient(Map<String, String> connectorConfig) { | ||
public SnowflakeStreamingIngestClient createClient( | ||
StreamingClientProperties streamingClientProperties) { | ||
LOGGER.info("Initializing Streaming Client..."); | ||
|
||
// get streaming properties from config | ||
Properties streamingClientProps = new Properties(); | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. this and getNewClientName is moved into StreamingClientProperties.java |
||
streamingClientProps.putAll( | ||
StreamingUtils.convertConfigForStreamingClient(new HashMap<>(connectorConfig))); | ||
|
||
try { | ||
// Override only if bdec version is explicitly set in config, default to the version set | ||
// inside Ingest SDK | ||
Map<String, Object> parameterOverrides = new HashMap<>(); | ||
Optional<String> snowpipeStreamingBdecVersion = | ||
Optional.ofNullable(connectorConfig.get(SNOWPIPE_STREAMING_FILE_VERSION)); | ||
snowpipeStreamingBdecVersion.ifPresent( | ||
overriddenValue -> { | ||
LOGGER.info("Config is overridden for {} ", SNOWPIPE_STREAMING_FILE_VERSION); | ||
parameterOverrides.put(BLOB_FORMAT_VERSION, overriddenValue); | ||
}); | ||
|
||
String clientName = this.getNewClientName(connectorConfig); | ||
|
||
SnowflakeStreamingIngestClient createdClient = | ||
SnowflakeStreamingIngestClientFactory.builder(clientName) | ||
.setProperties(streamingClientProps) | ||
.setParameterOverrides(parameterOverrides) | ||
SnowflakeStreamingIngestClientFactory.builder( | ||
streamingClientProperties.clientName + "_" + createdClientId.getAndIncrement()) | ||
.setProperties(streamingClientProperties.clientProperties) | ||
.setParameterOverrides(streamingClientProperties.parameterOverrides) | ||
.build(); | ||
|
||
LOGGER.info("Successfully initialized Streaming Client:{}", clientName); | ||
LOGGER.info( | ||
"Successfully initialized Streaming Client:{} with properties {}", | ||
streamingClientProperties.clientName, | ||
streamingClientProperties.getLoggableClientProperties()); | ||
|
||
return createdClient; | ||
} catch (SFException ex) { | ||
|
@@ -115,11 +92,4 @@ public void closeClient(SnowflakeStreamingIngestClient client) { | |
LOGGER.error(Utils.getExceptionMessage("Failure closing Streaming client", e)); | ||
} | ||
} | ||
|
||
private String getNewClientName(Map<String, String> connectorConfig) { | ||
return STREAMING_CLIENT_PREFIX_NAME | ||
+ connectorConfig.getOrDefault(Utils.NAME, TEST_CLIENT_NAME) | ||
+ "_" | ||
+ createdClientId.getAndIncrement(); | ||
} | ||
} |
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,134 @@ | ||
/* | ||
* Copyright (c) 2023 Snowflake Inc. All rights reserved. | ||
* | ||
* Licensed 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 com.snowflake.kafka.connector.internal.streaming; | ||
|
||
import static com.snowflake.kafka.connector.SnowflakeSinkConnectorConfig.SNOWPIPE_STREAMING_FILE_VERSION; | ||
import static net.snowflake.ingest.utils.ParameterProvider.BLOB_FORMAT_VERSION; | ||
|
||
import com.snowflake.kafka.connector.Utils; | ||
import com.snowflake.kafka.connector.internal.KCLogger; | ||
import java.util.HashMap; | ||
import java.util.List; | ||
import java.util.Map; | ||
import java.util.Optional; | ||
import java.util.Properties; | ||
import java.util.stream.Collectors; | ||
import java.util.stream.Stream; | ||
import net.snowflake.ingest.utils.Constants; | ||
|
||
/** | ||
* Object to convert and store properties for {@link | ||
* net.snowflake.ingest.streaming.SnowflakeStreamingIngestClient}. This object is used to compare | ||
* equality between clients in {@link StreamingClientProvider}. | ||
*/ | ||
public class StreamingClientProperties { | ||
public static final String STREAMING_CLIENT_PREFIX_NAME = "KC_CLIENT_"; | ||
public static final String DEFAULT_CLIENT_NAME = "DEFAULT_CLIENT"; | ||
|
||
private static final KCLogger LOGGER = new KCLogger(StreamingClientProperties.class.getName()); | ||
|
||
// contains converted config properties that are loggable (not PII data) | ||
public static final List<String> LOGGABLE_STREAMING_CONFIG_PROPERTIES = | ||
Stream.of( | ||
Constants.ACCOUNT_URL, | ||
Constants.ROLE, | ||
Constants.USER, | ||
StreamingUtils.STREAMING_CONSTANT_AUTHORIZATION_TYPE) | ||
.collect(Collectors.toList()); | ||
|
||
public final Properties clientProperties; | ||
public final String clientName; | ||
public final Map<String, Object> parameterOverrides; | ||
|
||
/** | ||
* Creates non-null properties, client name and parameter overrides for the {@link | ||
* net.snowflake.ingest.streaming.SnowflakeStreamingIngestClient} from the given connectorConfig | ||
* Properties are created by {@link StreamingUtils#convertConfigForStreamingClient(Map)} and are a | ||
* subset of the given connector configuration | ||
* | ||
* @param connectorConfig Given connector configuration. Null configs are treated as an empty map | ||
*/ | ||
public StreamingClientProperties(Map<String, String> connectorConfig) { | ||
// treat null config as empty config | ||
if (connectorConfig == null) { | ||
LOGGER.warn( | ||
"Creating empty streaming client properties because given connector config was empty"); | ||
connectorConfig = new HashMap<>(); | ||
} | ||
|
||
this.clientProperties = StreamingUtils.convertConfigForStreamingClient(connectorConfig); | ||
|
||
this.clientName = | ||
STREAMING_CLIENT_PREFIX_NAME | ||
+ connectorConfig.getOrDefault(Utils.NAME, DEFAULT_CLIENT_NAME); | ||
|
||
// Override only if bdec version is explicitly set in config, default to the version set | ||
// inside Ingest SDK | ||
this.parameterOverrides = new HashMap<>(); | ||
Optional<String> snowpipeStreamingBdecVersion = | ||
Optional.ofNullable(connectorConfig.get(SNOWPIPE_STREAMING_FILE_VERSION)); | ||
snowpipeStreamingBdecVersion.ifPresent( | ||
overriddenValue -> { | ||
LOGGER.info("Config is overridden for {} ", SNOWPIPE_STREAMING_FILE_VERSION); | ||
parameterOverrides.put(BLOB_FORMAT_VERSION, overriddenValue); | ||
}); | ||
} | ||
|
||
/** | ||
* Gets the loggable properties, see {@link | ||
* StreamingClientProperties#LOGGABLE_STREAMING_CONFIG_PROPERTIES} | ||
* | ||
* @return A formatted string with the loggable properties | ||
*/ | ||
public String getLoggableClientProperties() { | ||
return this.clientProperties == null | this.clientProperties.isEmpty() | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. can There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. it shouldn't ever be null or empty, but I added the check just in case |
||
? "" | ||
: this.clientProperties.entrySet().stream() | ||
.filter( | ||
propKvp -> | ||
LOGGABLE_STREAMING_CONFIG_PROPERTIES.stream() | ||
.anyMatch(propKvp.getKey().toString()::equalsIgnoreCase)) | ||
.collect(Collectors.toList()) | ||
.toString(); | ||
} | ||
|
||
/** | ||
* Determines equality between StreamingClientProperties by only looking at the parsed | ||
* clientProperties. This is used in {@link StreamingClientProvider} to determine equality in | ||
* registered clients | ||
* | ||
* @param other other object to determine equality | ||
* @return if the given object's clientProperties exists and is equal | ||
*/ | ||
@Override | ||
public boolean equals(Object other) { | ||
return other.getClass().equals(StreamingClientProperties.class) | ||
& ((StreamingClientProperties) other).clientProperties.equals(this.clientProperties); | ||
} | ||
|
||
/** | ||
* Creates the hashcode for this object from the clientProperties. This is used in {@link | ||
* StreamingClientProvider} to determine equality in registered clients | ||
* | ||
* @return the clientProperties' hashcode | ||
*/ | ||
@Override | ||
public int hashCode() { | ||
return this.clientProperties.hashCode(); | ||
} | ||
} |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Added this back (was removed in previous PR iterations) in case removing it causes concurrency issues with client naming