Skip to content

Commit 68d5c8e

Browse files
committed
Allow configuration of Cassandra's host selection policy (#4069)
Signed-off-by: ChinYing-Li <[email protected]>
1 parent cb8ed0e commit 68d5c8e

File tree

3 files changed

+22
-0
lines changed

3 files changed

+22
-0
lines changed

CHANGELOG.md

+1
Original file line numberDiff line numberDiff line change
@@ -24,6 +24,7 @@
2424
* [ENHANCEMENT] Distributor: reduce CPU and memory when an high number of errors are returned by the distributor on the write path. #3990
2525
* [ENHANCEMENT] Put metric before label value in the "label value too long" error message. #4018
2626
* [ENHANCEMENT] Allow use of `y|w|d` suffixes for duration related limits and per-tenant limits. #4044
27+
* [ENHANCEMENT] Allow configuration of Cassandra's host selection policy. (#4069)
2728
* [BUGFIX] Ruler-API: fix bug where `/api/v1/rules/<namespace>/<group_name>` endpoint return `400` instead of `404`. #4013
2829
* [BUGFIX] Distributor: reverted changes done to rate limiting in #3825. #3948
2930
* [BUGFIX] Ingester: Fix race condition when opening and closing tsdb concurrently. #3959

docs/configuration/config-file-reference.md

+5
Original file line numberDiff line numberDiff line change
@@ -2961,6 +2961,11 @@ cassandra:
29612961
# CLI flag: -cassandra.host-verification
29622962
[host_verification: <boolean> | default = true]
29632963

2964+
# Policy for selecting Cassandra host. Supported values are: round-robin,
2965+
# token-aware.
2966+
# CLI flag: -cassandra.host-selection-policy
2967+
[host_selection_policy: <string> | default = "round-robin"]
2968+
29642969
# Path to certificate file to verify the peer.
29652970
# CLI flag: -cassandra.ca-path
29662971
[CA_path: <string> | default = ""]

pkg/chunk/cassandra/storage_client.go

+16
Original file line numberDiff line numberDiff line change
@@ -33,6 +33,7 @@ type Config struct {
3333
DisableInitialHostLookup bool `yaml:"disable_initial_host_lookup"`
3434
SSL bool `yaml:"SSL"`
3535
HostVerification bool `yaml:"host_verification"`
36+
HostSelectionPolicy string `yaml:"host_selection_policy"`
3637
CAPath string `yaml:"CA_path"`
3738
CertPath string `yaml:"tls_cert_path"`
3839
KeyPath string `yaml:"tls_key_path"`
@@ -53,6 +54,11 @@ type Config struct {
5354
TableOptions string `yaml:"table_options"`
5455
}
5556

57+
const (
58+
HostPolicyRoundRobin = "round-robin"
59+
HostPolicyTokenAware = "token-aware"
60+
)
61+
5662
// RegisterFlags adds the flags required to config this to the given FlagSet
5763
func (cfg *Config) RegisterFlags(f *flag.FlagSet) {
5864
f.StringVar(&cfg.Addresses, "cassandra.addresses", "", "Comma-separated hostnames or IPs of Cassandra instances.")
@@ -63,6 +69,7 @@ func (cfg *Config) RegisterFlags(f *flag.FlagSet) {
6369
f.BoolVar(&cfg.DisableInitialHostLookup, "cassandra.disable-initial-host-lookup", false, "Instruct the cassandra driver to not attempt to get host info from the system.peers table.")
6470
f.BoolVar(&cfg.SSL, "cassandra.ssl", false, "Use SSL when connecting to cassandra instances.")
6571
f.BoolVar(&cfg.HostVerification, "cassandra.host-verification", true, "Require SSL certificate validation.")
72+
f.StringVar(&cfg.HostSelectionPolicy, "cassandra.host-selection-policy", HostPolicyRoundRobin, "Policy for selecting Cassandra host. Supported values are: round-robin, token-aware.")
6673
f.StringVar(&cfg.CAPath, "cassandra.ca-path", "", "Path to certificate file to verify the peer.")
6774
f.StringVar(&cfg.CertPath, "cassandra.tls-cert-path", "", "Path to certificate file used by TLS.")
6875
f.StringVar(&cfg.KeyPath, "cassandra.tls-key-path", "", "Path to private key file used by TLS.")
@@ -180,6 +187,15 @@ func (cfg *Config) setClusterConfig(cluster *gocql.ClusterConfig) error {
180187
}
181188
}
182189
}
190+
191+
if cfg.HostSelectionPolicy == HostPolicyRoundRobin {
192+
cluster.PoolConfig.HostSelectionPolicy = gocql.RoundRobinHostPolicy()
193+
} else if cfg.HostSelectionPolicy == HostPolicyTokenAware {
194+
cluster.PoolConfig.HostSelectionPolicy = gocql.TokenAwareHostPolicy(gocql.RoundRobinHostPolicy())
195+
} else {
196+
return errors.New("Unknown host selection policy")
197+
}
198+
183199
if cfg.Auth {
184200
password := cfg.Password.Value
185201
if cfg.PasswordFile != "" {

0 commit comments

Comments
 (0)