-
Notifications
You must be signed in to change notification settings - Fork 3.8k
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
67067: server: require admin role to access node status r=bdarnell a=knz Release note (security update): The node status retrieval endpoints over HTTP (`/_status/nodes`, `/_status/nodes/<N>` and the web UI `/#/reports/nodes`) have been updated to require the `admin` role from the requesting user. This ensures that operational details such as network addresses and command-line flags do not leak to unprivileged users. 67733: colexecbase: extend support of casts r=yuzefovich a=yuzefovich Addresses: #48135 See individual commits for details. After this PR we only need to add more casts between natively supported types. 67768: sql, server: add skeleton TokenBucket connector and tenant resource limits configuration APIs r=RaduBerinde a=RaduBerinde This PR is a scaled back version of #67508 where we don't use the system table at all. It's meant to put some of the infrastructure pieces in place and provide a stub API for reconfiguration. The plan is to add consumption metrics on top of this soon so that CC can develop in parallel. --- #### server: add TokenBucket connector API This change adds the TokenBucket API proposed in the RFC (#66436), a stub implementation and client for it, and the corresponding KV connector interface. The client and server-side code lives in ccl/multitenantccl/tenantcostclient and tenantcostserver. Release note: None #### sql: tenant resource limits configuration API This commit adds a `crdb_internal.update_tenant_resource_limits` internal SQL function (to be used by the system tenant) which updates the token bucket configuration for a specific tenant. Release note: None 67840: sql: add test for creating stats on tables with expression indexes r=mgartner a=mgartner Release note: None Co-authored-by: Raphael 'kena' Poss <[email protected]> Co-authored-by: Yahor Yuzefovich <[email protected]> Co-authored-by: Radu Berinde <[email protected]> Co-authored-by: Marcus Gartner <[email protected]>
- Loading branch information
Showing
63 changed files
with
8,865 additions
and
3,772 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,25 @@ | ||
# LogicTest: !3node-tenant | ||
|
||
query error tenant "13" does not exist | ||
SELECT crdb_internal.update_tenant_resource_limits(13, 1000, 100, 0, now(), 0) | ||
|
||
query I | ||
SELECT crdb_internal.create_tenant(5) | ||
---- | ||
5 | ||
|
||
# TODO(radu): inspect internal tenant_usage state. | ||
|
||
statement ok | ||
SELECT crdb_internal.update_tenant_resource_limits(5, 1000, 100, 0, now(), 0) | ||
|
||
# TODO(radu): inspect internal tenant_usage state. | ||
|
||
# Note this just marks the tenant as dropped but does not call GC. | ||
query I | ||
SELECT crdb_internal.destroy_tenant(5) | ||
---- | ||
5 | ||
|
||
query error tenant "5" is not active | ||
SELECT crdb_internal.update_tenant_resource_limits(5, 1000, 100, 0, now(), 0) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,12 @@ | ||
load("@io_bazel_rules_go//go:def.bzl", "go_library") | ||
|
||
go_library( | ||
name = "multitenantccl", | ||
srcs = ["doc.go"], | ||
importpath = "github.com/cockroachdb/cockroach/pkg/ccl/multitenantccl", | ||
visibility = ["//visibility:public"], | ||
deps = [ | ||
"//pkg/ccl/multitenantccl/tenantcostclient", | ||
"//pkg/ccl/multitenantccl/tenantcostserver", | ||
], | ||
) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,15 @@ | ||
// Copyright 2021 The Cockroach Authors. | ||
// | ||
// Licensed as a CockroachDB Enterprise file under the Cockroach Community | ||
// License (the "License"); you may not use this file except in compliance with | ||
// the License. You may obtain a copy of the License at | ||
// | ||
// https://github.com/cockroachdb/cockroach/blob/master/licenses/CCL.txt | ||
|
||
package multitenantccl | ||
|
||
import ( | ||
// Imports for the CCL init hooks. | ||
_ "github.com/cockroachdb/cockroach/pkg/ccl/multitenantccl/tenantcostclient" | ||
_ "github.com/cockroachdb/cockroach/pkg/ccl/multitenantccl/tenantcostserver" | ||
) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,17 @@ | ||
load("@io_bazel_rules_go//go:def.bzl", "go_library") | ||
|
||
go_library( | ||
name = "tenantcostclient", | ||
srcs = ["tenant_side.go"], | ||
importpath = "github.com/cockroachdb/cockroach/pkg/ccl/multitenantccl/tenantcostclient", | ||
visibility = ["//visibility:public"], | ||
deps = [ | ||
"//pkg/kv/kvclient/kvtenant", | ||
"//pkg/multitenant", | ||
"//pkg/roachpb:with-mocks", | ||
"//pkg/server", | ||
"//pkg/util/log", | ||
"//pkg/util/stop", | ||
"@com_github_cockroachdb_errors//:errors", | ||
], | ||
) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,80 @@ | ||
// Copyright 2021 The Cockroach Authors. | ||
// | ||
// Licensed as a CockroachDB Enterprise file under the Cockroach Community | ||
// License (the "License"); you may not use this file except in compliance with | ||
// the License. You may obtain a copy of the License at | ||
// | ||
// https://github.com/cockroachdb/cockroach/blob/master/licenses/CCL.txt | ||
|
||
package tenantcostclient | ||
|
||
import ( | ||
"context" | ||
"time" | ||
|
||
"github.com/cockroachdb/cockroach/pkg/kv/kvclient/kvtenant" | ||
"github.com/cockroachdb/cockroach/pkg/multitenant" | ||
"github.com/cockroachdb/cockroach/pkg/roachpb" | ||
"github.com/cockroachdb/cockroach/pkg/server" | ||
"github.com/cockroachdb/cockroach/pkg/util/log" | ||
"github.com/cockroachdb/cockroach/pkg/util/stop" | ||
"github.com/cockroachdb/errors" | ||
) | ||
|
||
// NewTenantSideCostController creates an object which implements the | ||
// server.TenantSideCostController interface. | ||
func NewTenantSideCostController( | ||
ctx context.Context, tenantID roachpb.TenantID, provider kvtenant.TokenBucketProvider, | ||
) (multitenant.TenantSideCostController, error) { | ||
if tenantID == roachpb.SystemTenantID { | ||
return nil, errors.AssertionFailedf("cost controller can't be used for system tenant") | ||
} | ||
return &tenantSideCostController{ | ||
tenantID: tenantID, | ||
provider: provider, | ||
}, nil | ||
} | ||
|
||
func init() { | ||
server.NewTenantSideCostController = NewTenantSideCostController | ||
} | ||
|
||
type tenantSideCostController struct { | ||
tenantID roachpb.TenantID | ||
provider kvtenant.TokenBucketProvider | ||
} | ||
|
||
var _ multitenant.TenantSideCostController = (*tenantSideCostController)(nil) | ||
|
||
// Start is part of multitenant.TenantSideCostController. | ||
func (c *tenantSideCostController) Start(ctx context.Context, stopper *stop.Stopper) error { | ||
return stopper.RunAsyncTask(ctx, "cost-controller", func(ctx context.Context) { | ||
c.mainLoop(ctx, stopper) | ||
}) | ||
} | ||
|
||
func (c *tenantSideCostController) mainLoop(ctx context.Context, stopper *stop.Stopper) { | ||
ticker := time.NewTicker(10 * time.Second) | ||
defer ticker.Stop() | ||
|
||
for { | ||
select { | ||
case <-ticker.C: | ||
req := roachpb.TokenBucketRequest{ | ||
ConsumptionSinceLastRequest: roachpb.TokenBucketRequest_Consumption{ | ||
// Report a dummy 1 RU consumption each time. | ||
RU: 1, | ||
SQLPodCPUSeconds: 1, | ||
}, | ||
} | ||
_, err := c.provider.TokenBucket(ctx, &req) | ||
if err != nil { | ||
log.Warningf(ctx, "TokenBucket error: %v", err) | ||
} | ||
|
||
case <-stopper.ShouldQuiesce(): | ||
// TODO(radu): send one last request to update consumption. | ||
return | ||
} | ||
} | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,26 @@ | ||
load("@io_bazel_rules_go//go:def.bzl", "go_library") | ||
|
||
go_library( | ||
name = "tenantcostserver", | ||
srcs = [ | ||
"configure.go", | ||
"server.go", | ||
"token_bucket.go", | ||
], | ||
importpath = "github.com/cockroachdb/cockroach/pkg/ccl/multitenantccl/tenantcostserver", | ||
visibility = ["//visibility:public"], | ||
deps = [ | ||
"//pkg/ccl/multitenantccl/tenantcostserver/tenanttokenbucket", | ||
"//pkg/kv", | ||
"//pkg/multitenant", | ||
"//pkg/roachpb:with-mocks", | ||
"//pkg/server", | ||
"//pkg/sql", | ||
"//pkg/sql/pgwire/pgcode", | ||
"//pkg/sql/pgwire/pgerror", | ||
"//pkg/sql/sem/tree", | ||
"//pkg/sql/sessiondata", | ||
"//pkg/util/timeutil", | ||
"@com_github_cockroachdb_errors//:errors", | ||
], | ||
) |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,63 @@ | ||
// Copyright 2021 The Cockroach Authors. | ||
// | ||
// Licensed as a CockroachDB Enterprise file under the Cockroach Community | ||
// License (the "License"); you may not use this file except in compliance with | ||
// the License. You may obtain a copy of the License at | ||
// | ||
// https://github.com/cockroachdb/cockroach/blob/master/licenses/CCL.txt | ||
|
||
package tenantcostserver | ||
|
||
import ( | ||
"context" | ||
"time" | ||
|
||
"github.com/cockroachdb/cockroach/pkg/kv" | ||
"github.com/cockroachdb/cockroach/pkg/roachpb" | ||
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgcode" | ||
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgerror" | ||
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree" | ||
"github.com/cockroachdb/cockroach/pkg/sql/sessiondata" | ||
"github.com/cockroachdb/cockroach/pkg/util/timeutil" | ||
"github.com/cockroachdb/errors" | ||
) | ||
|
||
// ReconfigureTokenBucket updates a tenant's token bucket settings. It is part | ||
// of the TenantUsageServer interface; see that for more details. | ||
func (s *instance) ReconfigureTokenBucket( | ||
ctx context.Context, | ||
txn *kv.Txn, | ||
tenantID roachpb.TenantID, | ||
availableRU float64, | ||
refillRate float64, | ||
maxBurstRU float64, | ||
asOf time.Time, | ||
asOfConsumedRequestUnits float64, | ||
) error { | ||
row, err := s.executor.QueryRowEx( | ||
ctx, "check-tenant", txn, sessiondata.NodeUserSessionDataOverride, | ||
`SELECT active FROM system.tenants WHERE id = $1`, tenantID.ToUint64(), | ||
) | ||
if err != nil { | ||
return err | ||
} | ||
if row == nil { | ||
return pgerror.Newf(pgcode.UndefinedObject, "tenant %q does not exist", tenantID) | ||
} | ||
if active := *row[0].(*tree.DBool); !active { | ||
return errors.Errorf("tenant %q is not active", tenantID) | ||
} | ||
state, err := readTenantUsageState(ctx, s.executor, txn, tenantID) | ||
if err != nil { | ||
return err | ||
} | ||
state.Seq++ | ||
state.Bucket.Reconfigure( | ||
availableRU, refillRate, maxBurstRU, asOf, asOfConsumedRequestUnits, | ||
timeutil.Now(), state.Consumption.RU, | ||
) | ||
if err := updateTenantUsageState(ctx, s.executor, txn, tenantID, state); err != nil { | ||
return err | ||
} | ||
return nil | ||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,38 @@ | ||
// Copyright 2021 The Cockroach Authors. | ||
// | ||
// Licensed as a CockroachDB Enterprise file under the Cockroach Community | ||
// License (the "License"); you may not use this file except in compliance with | ||
// the License. You may obtain a copy of the License at | ||
// | ||
// https://github.com/cockroachdb/cockroach/blob/master/licenses/CCL.txt | ||
|
||
package tenantcostserver | ||
|
||
import ( | ||
"github.com/cockroachdb/cockroach/pkg/kv" | ||
"github.com/cockroachdb/cockroach/pkg/multitenant" | ||
"github.com/cockroachdb/cockroach/pkg/server" | ||
"github.com/cockroachdb/cockroach/pkg/sql" | ||
) | ||
|
||
type instance struct { | ||
db *kv.DB | ||
executor *sql.InternalExecutor | ||
} | ||
|
||
func newInstance(db *kv.DB, executor *sql.InternalExecutor) *instance { | ||
return &instance{ | ||
db: db, | ||
executor: executor, | ||
} | ||
} | ||
|
||
var _ multitenant.TenantUsageServer = (*instance)(nil) | ||
|
||
func init() { | ||
server.NewTenantUsageServer = func( | ||
db *kv.DB, executor *sql.InternalExecutor, | ||
) multitenant.TenantUsageServer { | ||
return newInstance(db, executor) | ||
} | ||
} |
Oops, something went wrong.