yifan-c commented on code in PR #211: URL: https://github.com/apache/cassandra-sidecar/pull/211#discussion_r2004374489
########## conf/sidecar.yaml: ########## @@ -121,20 +121,37 @@ sidecar: replication_strategy: SimpleStrategy replication_factor: 1 # The TTL in seconds used to insert entries into the sidecar_lease schema - lease_schema_ttl: 2m + lease_schema_ttl: 5m coordination: # Captures configuration parameters for the task that performs the cluster lease claim process cluster_lease_claim: + # The name of the strategy used to determine the electorate membership + # Out of the box Sidecar provides the MostReplicatedKeyspaceTokenZeroElectorateMembership, and + # SidecarInternalTokenZeroElectorateMembership implementations. + # - MostReplicatedKeyspaceTokenZeroElectorateMembership the current Sidecar will be determined to be part + # of the electorate iff one of the Cassandra instances it + # manages owns token 0 for the user keyspace that has the + # highest replication factor. If multiple keyspaces have + # the highest replication factor, the keyspace to be used + # is decided by the keyspace with the name that sorts + # first in the lexicographic sort order. If no user + # keyspaces are created, the internal sidecar keyspace will + # be used. + # - SidecarInternalTokenZeroElectorateMembership the current Sidecar will be determined to be part of the + # electorate iff one of the Cassandra instances it manages + # owns token {@code 0} for the {@code sidecar_internal} + # keyspace. + electorate_membership_strategy: MostReplicatedKeyspaceTokenZeroElectorateMembership Review Comment: What is the default value if the new field `electorate_membership_strategy` is not configured? ########## examples/conf/sidecar-ccm.yaml: ########## @@ -277,6 +307,19 @@ access_control: # # other options are, io.vertx.ext.auth.mtls.impl.SpiffeIdentityExtractor. certificate_identity_extractor: org.apache.cassandra.sidecar.acl.authentication.CassandraIdentityExtractor + # JwtAuthenticationHandlerFactory adds support to authenticate users with their JWT tokens. It also includes + # supports for OpenID discovery. + - class_name: org.apache.cassandra.sidecar.acl.authentication.JwtAuthenticationHandlerFactory + parameters: + # To selectively enable or disable JWT authentication + enabled: false + # Site for sidecar to dynamically retrieve the configuration information of an OpenID provider, without + # having to manually configure settings like issuer etc. + site: https://authorization.com Review Comment: Thanks for adding them back. It is easy to forgot update this file. We should think of "copying" from `conf/sidecar.yaml` in the future. ########## conf/sidecar.yaml: ########## @@ -121,20 +121,37 @@ sidecar: replication_strategy: SimpleStrategy replication_factor: 1 # The TTL in seconds used to insert entries into the sidecar_lease schema - lease_schema_ttl: 2m + lease_schema_ttl: 5m coordination: # Captures configuration parameters for the task that performs the cluster lease claim process cluster_lease_claim: + # The name of the strategy used to determine the electorate membership + # Out of the box Sidecar provides the MostReplicatedKeyspaceTokenZeroElectorateMembership, and + # SidecarInternalTokenZeroElectorateMembership implementations. + # - MostReplicatedKeyspaceTokenZeroElectorateMembership the current Sidecar will be determined to be part + # of the electorate iff one of the Cassandra instances it + # manages owns token 0 for the user keyspace that has the + # highest replication factor. If multiple keyspaces have + # the highest replication factor, the keyspace to be used + # is decided by the keyspace with the name that sorts + # first in the lexicographic sort order. If no user + # keyspaces are created, the internal sidecar keyspace will + # be used. + # - SidecarInternalTokenZeroElectorateMembership the current Sidecar will be determined to be part of the + # electorate iff one of the Cassandra instances it manages + # owns token {@code 0} for the {@code sidecar_internal} + # keyspace. + electorate_membership_strategy: MostReplicatedKeyspaceTokenZeroElectorateMembership # Whether the process is enabled enabled: true # The initial delay for the first execution of the cluster lease claim process task after being # scheduled or rescheduled. # The minimum value for the initial delay is 0ms. - initial_delay: 1s Review Comment: > The minimum value for the initial delay is 0ms. Is it true? Should the minimum be 0? It will cause a lot of cas contention. ########## server/src/main/java/org/apache/cassandra/sidecar/coordination/AbstractTokenZeroOfKeyspaceElectorateMembership.java: ########## @@ -0,0 +1,149 @@ +/* + * 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.cassandra.sidecar.coordination; + +import java.math.BigInteger; +import java.net.InetSocketAddress; +import java.util.HashSet; +import java.util.List; +import java.util.Set; + +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import org.apache.cassandra.sidecar.cluster.CassandraAdapterDelegate; +import org.apache.cassandra.sidecar.cluster.instance.InstanceMetadata; +import org.apache.cassandra.sidecar.common.response.NodeSettings; +import org.apache.cassandra.sidecar.common.response.TokenRangeReplicasResponse; +import org.apache.cassandra.sidecar.common.server.StorageOperations; +import org.apache.cassandra.sidecar.common.server.data.Name; +import org.apache.cassandra.sidecar.common.server.utils.StringUtils; +import org.apache.cassandra.sidecar.exceptions.CassandraUnavailableException; +import org.apache.cassandra.sidecar.utils.InstanceMetadataFetcher; + +/** + * Provides common functionality for {@link ElectorateMembership} implementations + * that rely on token zero replication of a keyspace to determine eligibility. + */ +public abstract class AbstractTokenZeroOfKeyspaceElectorateMembership implements ElectorateMembership +{ + private static final Logger LOGGER = LoggerFactory.getLogger(AbstractTokenZeroOfKeyspaceElectorateMembership.class); + protected final InstanceMetadataFetcher instanceMetadataFetcher; + + public AbstractTokenZeroOfKeyspaceElectorateMembership(InstanceMetadataFetcher instanceMetadataFetcher) + { + this.instanceMetadataFetcher = instanceMetadataFetcher; + } + + /** + * {@inheritDoc} + */ + @Override + public boolean isMember() + { + Set<String> localInstancesHostsAndPorts = collectLocalInstancesHostsAndPorts(); + if (localInstancesHostsAndPorts.isEmpty()) + { + // Unable to retrieve local instances, maybe all Cassandra connections are down? + return false; + } + + String keyspace = keyspaceToDetermineElectorateMembership(); + if (keyspace == null) + { + // pre-checks failed + return false; + } + LOGGER.debug("Using keyspace={} to determine electorate membership", keyspace); + + TokenRangeReplicasResponse tokenRangeReplicas = instanceMetadataFetcher.callOnFirstAvailableInstance(instance -> { + CassandraAdapterDelegate delegate = instance.delegate(); + StorageOperations operations = delegate.storageOperations(); + NodeSettings nodeSettings = delegate.nodeSettings(); + return operations.tokenRangeReplicas(new Name(keyspace), nodeSettings.partitioner()); + }); + + return anyInstanceOwnsTokenZero(tokenRangeReplicas, localInstancesHostsAndPorts); + } + + /** + * @return the name of the keyspace that will be used to determine the electorate membership + */ + protected abstract String keyspaceToDetermineElectorateMembership(); + + Set<String> collectLocalInstancesHostsAndPorts() + { + Set<String> result = new HashSet<>(); + for (InstanceMetadata instance : instanceMetadataFetcher.allLocalInstances()) + { + try + { + InetSocketAddress address = instance.delegate().localStorageBroadcastAddress(); + result.add(StringUtils.cassandraFormattedHostAndPort(address)); + } + catch (CassandraUnavailableException exception) + { + // Log a warning message and continue + LOGGER.warn("Unable to determine local storage broadcast address for instance. instance={}", instance, exception); + } + } + return result; + } + + /** + * @param tokenRangeReplicas the token range replicas for a keyspace + * @param localInstancesHostAndPorts local instance(s) IP(s) and port(s) + * @return {@code true} if any of the local instances is a replica of token zero for a single keyspace, + * {@code false} otherwise + */ + boolean anyInstanceOwnsTokenZero(TokenRangeReplicasResponse tokenRangeReplicas, Set<String> localInstancesHostAndPorts) Review Comment: curious, why not `protected` for this and for `collectLocalInstancesHostsAndPorts` and `replicaOwnsTokenZero`? ########## server/src/main/java/org/apache/cassandra/sidecar/db/SidecarLeaseDatabaseAccessor.java: ########## @@ -76,20 +78,22 @@ public LeaseClaimResult extendLease(String currentOwner) */ public static class LeaseClaimResult { - public final boolean leaseAcquired; public final String currentOwner; - LeaseClaimResult(boolean leaseAcquired, String currentOwner) + LeaseClaimResult(String currentOwner) { - this.leaseAcquired = leaseAcquired; this.currentOwner = currentOwner; } static LeaseClaimResult from(ResultSet resultSet, String newOwner) { return resultSet.wasApplied() - ? new LeaseClaimResult(true, newOwner) - : new LeaseClaimResult(false, resultSet.one().getString("owner")); + ? new LeaseClaimResult(newOwner) + // In some rare cases, the resultSet will not contain the owner information + // even though the resultSet was not applied. This will translate into an + // IllegalArgumentException being thrown when trying to retrieve the non-existing + // owner string. This exception is left to be handled by the caller method + : new LeaseClaimResult(resultSet.one().getString("owner")); Review Comment: It is unclear to me what is the proper handling at the caller. Maybe handle exception here and set a new state, say "unknown". Caller can react on the "unknown" state, for example, query retry. ########## server/src/main/java/org/apache/cassandra/sidecar/modules/CoordinationModule.java: ########## @@ -70,6 +72,18 @@ ElectorateMembership electorateMembership(InstanceMetadataFetcher instanceMetada CQLSessionProvider cqlSessionProvider, SidecarConfiguration configuration) { - return new MostReplicatedKeyspaceTokenZeroElectorateMembership(instanceMetadataFetcher, cqlSessionProvider, configuration); + String strategy = configuration.serviceConfiguration() + .coordinationConfiguration() + .clusterLeaseClaimConfiguration() + .electorateMembershipStrategy(); + switch (strategy) + { + case "MostReplicatedKeyspaceTokenZeroElectorateMembership": + return new MostReplicatedKeyspaceTokenZeroElectorateMembership(instanceMetadataFetcher, cqlSessionProvider, configuration); + case "SidecarInternalTokenZeroElectorateMembership": + return new SidecarInternalTokenZeroElectorateMembership(instanceMetadataFetcher, configuration); + default: + throw new ConfigurationException("Invalid electorate membership strategy value '" + strategy + "'"); + } Review Comment: can we move this code to a factory class? Module code is hard to testable, and complex code should be tested. -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: pr-unsubscr...@cassandra.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org --------------------------------------------------------------------- To unsubscribe, e-mail: pr-unsubscr...@cassandra.apache.org For additional commands, e-mail: pr-h...@cassandra.apache.org