Skip to content
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

Implement stream connection for remote write #6580

Draft
wants to merge 2 commits into
base: master
Choose a base branch
from
Draft
Show file tree
Hide file tree
Changes from all commits
Commits
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
5 changes: 5 additions & 0 deletions docs/configuration/config-file-reference.md
Original file line number Diff line number Diff line change
Expand Up @@ -2657,6 +2657,11 @@ ha_tracker:
# CLI flag: -distributor.sign-write-requests
[sign_write_requests: <boolean> | default = false]

# EXPERIMENTAL: If enabled, distributor would use stream connection to send
# requests to ingesters.
# CLI flag: -distributor.use-stream-push
[use_stream_push: <boolean> | default = false]

ring:
kvstore:
# Backend storage to use for the ring. Supported values are: consul, etcd,
Expand Down
499 changes: 392 additions & 107 deletions pkg/cortexpb/cortex.pb.go

Large diffs are not rendered by default.

5 changes: 5 additions & 0 deletions pkg/cortexpb/cortex.proto
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,11 @@ message WriteRequest {
bool skip_label_name_validation = 1000; //set intentionally high to keep WriteRequest compatible with upstream Prometheus
}

message StreamWriteRequest {
string TenantID = 1;
WriteRequest Request = 2;
}

message WriteResponse {}

message TimeSeries {
Expand Down
35 changes: 23 additions & 12 deletions pkg/distributor/distributor.go
Original file line number Diff line number Diff line change
Expand Up @@ -4,7 +4,7 @@ import (
"context"
"flag"
"fmt"
io "io"
"io"
"net/http"
"sort"
"strings"
Expand Down Expand Up @@ -150,6 +150,7 @@ type Config struct {
ShardByAllLabels bool `yaml:"shard_by_all_labels"`
ExtendWrites bool `yaml:"extend_writes"`
SignWriteRequestsEnabled bool `yaml:"sign_write_requests"`
UseStreamPush bool `yaml:"use_stream_push"`

// Distributors ring
DistributorRing RingConfig `yaml:"ring"`
Expand Down Expand Up @@ -204,6 +205,7 @@ func (cfg *Config) RegisterFlags(f *flag.FlagSet) {
f.DurationVar(&cfg.ExtraQueryDelay, "distributor.extra-query-delay", 0, "Time to wait before sending more than the minimum successful query requests.")
f.BoolVar(&cfg.ShardByAllLabels, "distributor.shard-by-all-labels", false, "Distribute samples based on all labels, as opposed to solely by user and metric name.")
f.BoolVar(&cfg.SignWriteRequestsEnabled, "distributor.sign-write-requests", false, "EXPERIMENTAL: If enabled, sign the write request between distributors and ingesters.")
f.BoolVar(&cfg.UseStreamPush, "distributor.use-stream-push", false, "EXPERIMENTAL: If enabled, distributor would use stream connection to send requests to ingesters.")
f.StringVar(&cfg.ShardingStrategy, "distributor.sharding-strategy", util.ShardingStrategyDefault, fmt.Sprintf("The sharding strategy to use. Supported values are: %s.", strings.Join(supportedShardingStrategies, ", ")))
f.BoolVar(&cfg.ExtendWrites, "distributor.extend-writes", true, "Try writing to an additional ingester in the presence of an ingester not in the ACTIVE state. It is useful to disable this along with -ingester.unregister-on-shutdown=false in order to not spread samples to extra ingesters during rolling restarts with consistent naming.")
f.BoolVar(&cfg.ZoneResultsQuorumMetadata, "distributor.zone-results-quorum-metadata", false, "Experimental, this flag may change in the future. If zone awareness and this both enabled, when querying metadata APIs (labels names and values for now), only results from quorum number of zones will be included.")
Expand Down Expand Up @@ -242,7 +244,7 @@ const (
func New(cfg Config, clientConfig ingester_client.Config, limits *validation.Overrides, ingestersRing ring.ReadRing, canJoinDistributorsRing bool, reg prometheus.Registerer, log log.Logger) (*Distributor, error) {
if cfg.IngesterClientFactory == nil {
cfg.IngesterClientFactory = func(addr string) (ring_client.PoolClient, error) {
return ingester_client.MakeIngesterClient(addr, clientConfig)
return ingester_client.MakeIngesterClient(addr, clientConfig, cfg.UseStreamPush)
}
}

Expand Down Expand Up @@ -1134,20 +1136,29 @@ func (d *Distributor) send(ctx context.Context, ingester ring.InstanceDesc, time

c := h.(ingester_client.HealthAndIngesterClient)

req := cortexpb.PreallocWriteRequestFromPool()
req.Timeseries = timeseries
req.Metadata = metadata
req.Source = source

d.inflightClientRequests.Inc()
defer d.inflightClientRequests.Dec()

_, err = c.PushPreAlloc(ctx, req)
if d.cfg.UseStreamPush {
req := &cortexpb.WriteRequest{
Timeseries: timeseries,
Metadata: metadata,
Source: source,
}
_, err = c.PushStreamConnection(ctx, req)
} else {
req := cortexpb.PreallocWriteRequestFromPool()
req.Timeseries = timeseries
req.Metadata = metadata
req.Source = source

// We should not reuse the req in case of errors:
// See: https://github.com/grpc/grpc-go/issues/6355
if err == nil {
cortexpb.ReuseWriteRequest(req)
_, err = c.PushPreAlloc(ctx, req)

// We should not reuse the req in case of errors:
// See: https://github.com/grpc/grpc-go/issues/6355
if err == nil {
cortexpb.ReuseWriteRequest(req)
}
}

if len(metadata) > 0 {
Expand Down
4 changes: 4 additions & 0 deletions pkg/distributor/distributor_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -3307,6 +3307,10 @@ func (i *mockIngester) PushPreAlloc(ctx context.Context, in *cortexpb.PreallocWr
return i.Push(ctx, &in.WriteRequest, opts...)
}

func (i *mockIngester) PushStreamConnection(ctx context.Context, in *cortexpb.WriteRequest, opts ...grpc.CallOption) (*cortexpb.WriteResponse, error) {
return i.Push(ctx, in, opts...)
}

func (i *mockIngester) Push(ctx context.Context, req *cortexpb.WriteRequest, opts ...grpc.CallOption) (*cortexpb.WriteResponse, error) {
i.Lock()
defer i.Unlock()
Expand Down
117 changes: 114 additions & 3 deletions pkg/ingester/client/client.go
Original file line number Diff line number Diff line change
Expand Up @@ -3,15 +3,19 @@ package client
import (
"context"
"flag"
"fmt"
"io"

"github.com/cortexproject/cortex/pkg/cortexpb"
"github.com/cortexproject/cortex/pkg/tenant"
"github.com/cortexproject/cortex/pkg/util/grpcclient"
"github.com/cortexproject/cortex/pkg/util/grpcencoding/snappyblock"

"github.com/go-kit/log"
"github.com/pkg/errors"
"github.com/prometheus/client_golang/prometheus"
"github.com/prometheus/client_golang/prometheus/promauto"
"github.com/weaveworks/common/user"
"go.uber.org/atomic"
"google.golang.org/grpc"
"google.golang.org/grpc/health/grpc_health_v1"
Expand All @@ -31,6 +35,8 @@ var ingesterClientInflightPushRequests = promauto.NewGaugeVec(prometheus.GaugeOp

var errTooManyInflightPushRequests = errors.New("too many inflight push requests in ingester client")

const INGESTER_CLIENT_STREAM_WORKER_COUNT = 1024

// ClosableClientConn is grpc.ClientConnInterface with Close function
type ClosableClientConn interface {
grpc.ClientConnInterface
Expand All @@ -43,6 +49,15 @@ type HealthAndIngesterClient interface {
grpc_health_v1.HealthClient
Close() error
PushPreAlloc(ctx context.Context, in *cortexpb.PreallocWriteRequest, opts ...grpc.CallOption) (*cortexpb.WriteResponse, error)
PushStreamConnection(ctx context.Context, in *cortexpb.WriteRequest, opts ...grpc.CallOption) (*cortexpb.WriteResponse, error)
}

type streamWriteJob struct {
req *cortexpb.StreamWriteRequest
resp *cortexpb.WriteResponse
ctx context.Context
cancel context.CancelFunc
err error
}

type closableHealthAndIngesterClient struct {
Expand All @@ -53,6 +68,9 @@ type closableHealthAndIngesterClient struct {
maxInflightPushRequests int64
inflightRequests atomic.Int64
inflightPushRequests *prometheus.GaugeVec
streamPushChan chan *streamWriteJob
streamCtx context.Context
streamCancel context.CancelFunc
}

func (c *closableHealthAndIngesterClient) PushPreAlloc(ctx context.Context, in *cortexpb.PreallocWriteRequest, opts ...grpc.CallOption) (*cortexpb.WriteResponse, error) {
Expand All @@ -72,6 +90,32 @@ func (c *closableHealthAndIngesterClient) Push(ctx context.Context, in *cortexpb
})
}

func (c *closableHealthAndIngesterClient) PushStreamConnection(ctx context.Context, in *cortexpb.WriteRequest, opts ...grpc.CallOption) (*cortexpb.WriteResponse, error) {
return c.handlePushRequest(func() (*cortexpb.WriteResponse, error) {
tenantID, err := tenant.TenantID(ctx)
if err != nil {
return nil, err
}

streamReq := &cortexpb.StreamWriteRequest{
TenantID: tenantID,
Request: in,
}

reqCtx, reqCancel := context.WithCancel(ctx)
defer reqCancel()

job := &streamWriteJob{
req: streamReq,
ctx: reqCtx,
cancel: reqCancel,
}
c.streamPushChan <- job
<-reqCtx.Done()
return job.resp, job.err
})
}

func (c *closableHealthAndIngesterClient) handlePushRequest(mainFunc func() (*cortexpb.WriteResponse, error)) (*cortexpb.WriteResponse, error) {
currentInflight := c.inflightRequests.Inc()
c.inflightPushRequests.WithLabelValues(c.addr).Set(float64(currentInflight))
Expand All @@ -85,7 +129,7 @@ func (c *closableHealthAndIngesterClient) handlePushRequest(mainFunc func() (*co
}

// MakeIngesterClient makes a new IngesterClient
func MakeIngesterClient(addr string, cfg Config) (HealthAndIngesterClient, error) {
func MakeIngesterClient(addr string, cfg Config, useStreamConnection bool) (HealthAndIngesterClient, error) {
dialOpts, err := cfg.GRPCClientConfig.DialOption(grpcclient.Instrument(ingesterClientRequestDuration))
if err != nil {
return nil, err
Expand All @@ -94,21 +138,88 @@ func MakeIngesterClient(addr string, cfg Config) (HealthAndIngesterClient, error
if err != nil {
return nil, err
}
return &closableHealthAndIngesterClient{
streamCtx, streamCancel := context.WithCancel(context.Background())
c := &closableHealthAndIngesterClient{
IngesterClient: NewIngesterClient(conn),
HealthClient: grpc_health_v1.NewHealthClient(conn),
conn: conn,
addr: addr,
maxInflightPushRequests: cfg.MaxInflightPushRequests,
inflightPushRequests: ingesterClientInflightPushRequests,
}, nil
streamPushChan: make(chan *streamWriteJob, INGESTER_CLIENT_STREAM_WORKER_COUNT),
streamCtx: streamCtx,
streamCancel: streamCancel,
}
if useStreamConnection {
err = c.Run()
if err != nil {
return nil, err
}
}
return c, nil
}

func (c *closableHealthAndIngesterClient) Close() error {
c.inflightPushRequests.DeleteLabelValues(c.addr)
c.streamCancel()
return c.conn.Close()
}

func (c *closableHealthAndIngesterClient) Run() error {
var err error
for i := 0; i < INGESTER_CLIENT_STREAM_WORKER_COUNT; i++ {
workerCtx := user.InjectOrgID(c.streamCtx, fmt.Sprintf("stream-worker-%d", i))
go func() {
for {
select {
case <-workerCtx.Done():
return
default:
err = c.worker(workerCtx)
if err != nil {
return
}
}
}
}()
}
return err
}

func (c *closableHealthAndIngesterClient) worker(ctx context.Context) error {
stream, err := c.PushStream(ctx)
if err != nil {
return err
}
for {
select {
case <-ctx.Done():
return nil
case job := <-c.streamPushChan:
err = stream.Send(job.req)
if err == io.EOF {
Copy link
Contributor

@anna-tran anna-tran Feb 7, 2025

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

when does the error become EOF? when the stream is closed but we try to send the job anyways?

could we extract this comparison to a well-named helper method to make it more explicit about what EOF means in this case?

Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

I think so. I checked how stream connection got handled elsewhere. Just handled this in similar way.

job.resp = &cortexpb.WriteResponse{}
job.cancel()
return nil
}
if err != nil {
job.err = err
job.cancel()
continue
}
resp, err := stream.Recv()
if err == io.EOF {
job.resp = &cortexpb.WriteResponse{}
job.cancel()
return nil
}
job.resp = resp
job.err = err
job.cancel()
}
}
}

// Config is the configuration struct for the ingester client
type Config struct {
GRPCClientConfig grpcclient.ConfigWithHealthCheck `yaml:"grpc_client_config"`
Expand Down
5 changes: 5 additions & 0 deletions pkg/ingester/client/cortex_mock_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,11 @@ func (m *IngesterServerMock) Push(ctx context.Context, r *cortexpb.WriteRequest)
return args.Get(0).(*cortexpb.WriteResponse), args.Error(1)
}

func (m *IngesterServerMock) PushStream(srv Ingester_PushStreamServer) error {
args := m.Called(srv)
return args.Error(0)
}

func (m *IngesterServerMock) Query(ctx context.Context, r *QueryRequest) (*QueryResponse, error) {
args := m.Called(ctx, r)
return args.Get(0).(*QueryResponse), args.Error(1)
Expand Down
Loading
Loading