Skip to content

Commit

Permalink
sql,server: change indexusagestats subsystem to issue cluster RPC fanout
Browse files Browse the repository at this point in the history
Previously, indexusagestats subsystem can only read node-local index usage
stats. This commit introduces clusterindexusagestats package within the
indexusagestats subsystem to use RPC fanout infrastructure inside
statusServer to fetch cluster-wide index usage statistics.

Related issue: #64740

Release note: None
  • Loading branch information
Azhng committed Jul 14, 2021
1 parent 84f7aa0 commit 5692451
Show file tree
Hide file tree
Showing 19 changed files with 1,727 additions and 442 deletions.
47 changes: 47 additions & 0 deletions docs/generated/http/full.md
Original file line number Diff line number Diff line change
Expand Up @@ -3129,6 +3129,53 @@ Response object returned by ResetSQLStats.



## IndexUsageStatistics

`GET /_status/indexusagestatistics`



Support status: [reserved](#support-status)

#### Request Parameters




Request object for issuing IndexUsageStatistics request.


| Field | Type | Label | Description | Support status |
| ----- | ---- | ----- | ----------- | -------------- |
| node_id | [string](#cockroach.server.serverpb.IndexUsageStatisticsRequest-string) | | node_id is the ID of the node where the stats data shall be retrieved from. If this is left empty, the cluster-wide aggregated result will be returned. | [reserved](#support-status) |
| ordered_table_id | [bool](#cockroach.server.serverpb.IndexUsageStatisticsRequest-bool) | | ordered_table_id specifies that if the returned stats data will be ordered based on the table_id. | [reserved](#support-status) |
| ordered_index_id | [bool](#cockroach.server.serverpb.IndexUsageStatisticsRequest-bool) | | ordered_index_id specifies that if the returned stats data's index ID will be ordered. | [reserved](#support-status) |
| max_limit | [uint64](#cockroach.server.serverpb.IndexUsageStatisticsRequest-uint64) | | | [reserved](#support-status) |







#### Response Parameters




Response object returned by IndexUsageStatistics.


| Field | Type | Label | Description | Support status |
| ----- | ---- | ----- | ----------- | -------------- |
| statistics | [cockroach.sql.CollectedIndexUsageStatistics](#cockroach.server.serverpb.IndexUsageStatisticsResponse-cockroach.sql.CollectedIndexUsageStatistics) | repeated | | [reserved](#support-status) |







## RequestCA

`GET /_join/v1/ca`
Expand Down
1 change: 1 addition & 0 deletions pkg/base/testing_knobs.go
Original file line number Diff line number Diff line change
Expand Up @@ -38,4 +38,5 @@ type TestingKnobs struct {
JobsTestingKnobs ModuleTestingKnobs
BackupRestore ModuleTestingKnobs
MigrationManager ModuleTestingKnobs
IndexUsageStatsKnobs ModuleTestingKnobs
}
4 changes: 4 additions & 0 deletions pkg/server/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -20,6 +20,7 @@ go_library(
"drain.go",
"grpc_server.go",
"idle_monitor.go",
"index_usage_stats.go",
"init.go",
"init_handshake.go",
"loopback.go",
Expand Down Expand Up @@ -124,6 +125,7 @@ go_library(
"//pkg/sql/flowinfra",
"//pkg/sql/gcjob",
"//pkg/sql/gcjob/gcjobnotifier",
"//pkg/sql/idxusage",
"//pkg/sql/optionalnodeliveness",
"//pkg/sql/parser",
"//pkg/sql/pgwire",
Expand Down Expand Up @@ -260,6 +262,7 @@ go_test(
"drain_test.go",
"graphite_test.go",
"idle_monitor_test.go",
"index_usage_stats_test.go",
"init_handshake_test.go",
"intent_test.go",
"main_test.go",
Expand Down Expand Up @@ -320,6 +323,7 @@ go_test(
"//pkg/sql/catalog/dbdesc",
"//pkg/sql/catalog/descpb",
"//pkg/sql/execinfrapb",
"//pkg/sql/idxusage",
"//pkg/sql/sem/tree",
"//pkg/sql/sqlstats",
"//pkg/sql/tests",
Expand Down
134 changes: 134 additions & 0 deletions pkg/server/index_usage_stats.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,134 @@
// Copyright 2021 The Cockroach Authors.
//
// Use of this software is governed by the Business Source License
// included in the file licenses/BSL.txt.
//
// As of the Change Date specified in that file, in accordance with
// the Business Source License, use of this software will be governed
// by the Apache License, Version 2.0, included in the file
// licenses/APL.txt.

package server

import (
"context"
"fmt"

"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/server/serverpb"
"github.com/cockroachdb/cockroach/pkg/sql/idxusage"
"github.com/cockroachdb/errors"
"google.golang.org/grpc/codes"
"google.golang.org/grpc/status"
)

// IndexUsageStatistics is the GRPC handler for serving index usage statistics.
// If the NodeID in the request payload is left empty, the handler will issue
// a cluster-wide RPC fanout to aggregate all index usage statistics from all
// the nodes. If the NodeID is specified, then the handler will handle the
// request either locally (if the NodeID matches the current node's NodeID) or
// forward it to the correct node.
func (s *statusServer) IndexUsageStatistics(
ctx context.Context, req *serverpb.IndexUsageStatisticsRequest,
) (*serverpb.IndexUsageStatisticsResponse, error) {
ctx = propagateGatewayMetadata(ctx)
ctx = s.AnnotateCtx(ctx)

if _, err := s.privilegeChecker.requireViewActivityPermission(ctx); err != nil {
return nil, err
}

localReq := &serverpb.IndexUsageStatisticsRequest{
NodeID: "local",
OrderedTableID: req.OrderedTableID,
OrderedIndexID: req.OrderedIndexID,
}

if len(req.NodeID) > 0 {
requestedNodeID, local, err := s.parseNodeID(req.NodeID)
if err != nil {
return nil, status.Error(codes.InvalidArgument, err.Error())
}
if local {
statsReader :=
s.sqlServer.pgServer.SQLServer.GetLocalIndexStatisticsReader()
return serializeIndexUsageStats(req, statsReader)
}

statusServer, err := s.dialNode(ctx, requestedNodeID)
if err != nil {
return nil, err
}

return statusServer.IndexUsageStatistics(ctx, localReq)
}

// Creating a sink to aggregate all the information.
aggStats := s.sqlServer.pgServer.SQLServer.GetClusterIndexStatisticsProvider()
aggStats.Clear()

dialFn := func(ctx context.Context, nodeID roachpb.NodeID) (interface{}, error) {
client, err := s.dialNode(ctx, nodeID)
return client, err
}

fetchIndexUsageStats := func(ctx context.Context, client interface{}, _ roachpb.NodeID) (interface{}, error) {
statusClient := client.(serverpb.StatusClient)
return statusClient.IndexUsageStatistics(ctx, localReq)
}

aggFn := func(_ roachpb.NodeID, resp interface{}) {
stats := resp.(*serverpb.IndexUsageStatisticsResponse)
aggStats.BatchInsert(stats.Statistics)
}

var combinedError error
errFn := func(_ roachpb.NodeID, nodeFnError error) {
if nodeFnError != nil {
combinedError = errors.CombineErrors(combinedError, nodeFnError)
}
}

// It's unfortunate that we cannot use paginatedIterateNodes here because we
// need to aggregate all stats before returning. Returning a partial result
// yields an incorrect result.
if err := s.iterateNodes(ctx,
fmt.Sprintf("requesting index usage stats for node %s", req.NodeID),
dialFn, fetchIndexUsageStats, aggFn, errFn); err != nil {
return nil, err
}

return serializeIndexUsageStats(req, aggStats)
}

func serializeIndexUsageStats(
req *serverpb.IndexUsageStatisticsRequest, reader idxusage.Reader,
) (*serverpb.IndexUsageStatisticsResponse, error) {
indexStats := make([]roachpb.CollectedIndexUsageStatistics, 0)

var max *uint64
if req.GetMax() != nil {
maxLimit := req.GetMaxLimit()
max = &maxLimit
}

err := reader.ForEach(idxusage.IteratorOptions{
SortedTableID: req.OrderedTableID,
SortedIndexID: req.OrderedIndexID,
Max: max,
}, func(key *roachpb.IndexUsageKey, value *roachpb.IndexUsageStatistics) error {
indexStats = append(indexStats, roachpb.CollectedIndexUsageStatistics{
Key: *key,
Stats: *value,
})
return nil
})

if err != nil {
return nil, err
}

return &serverpb.IndexUsageStatisticsResponse{
Statistics: indexStats,
}, nil
}
Loading

0 comments on commit 5692451

Please sign in to comment.