Skip to content

Commit

Permalink
storage/report: don't deserialize zone configs over and over
Browse files Browse the repository at this point in the history
This patch is expected to speedup reports generation considerably by not
unmarshalling zone config protos for every range. Instead, the
report-geneating visitors now keep state around the zone that a visited
range is in and reuse that state if they're told that the following
range is in the same zone. The range iteration infrastructure was
enhanced to figure out when zones change from range to range and tell
that to the visitors.

visitRanges() now figures out that runs of consecutive ranges belong to
the same zone. This is done through a new zoneResolver struct, that's
optimized for the case where it's asked to resolve ranges in key order.

Without this patch, generating the reports was pinning a core for
minutes for a cluster with partitioning and 200k ranges. The profiles
showed that it's all zone config unmarshalling.

Fixes cockroachdb#41609

Release note (performance improvement): The performance of generating
the system.replication_* reports was greatly improved for large
clusters.
  • Loading branch information
andreimatei committed Oct 21, 2019
1 parent 207f5b2 commit 2e21595
Show file tree
Hide file tree
Showing 9 changed files with 583 additions and 134 deletions.
6 changes: 6 additions & 0 deletions pkg/base/zone.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,12 @@ package base
// sentinel.
type SubzoneID uint32

// ToSubzoneIndex turns a SubzoneID into the index corresponding to the correct
// Subzone within the parent zone's Subzones slice.
func (id SubzoneID) ToSubzoneIndex() int32 {
return int32(id) - 1
}

// SubzoneIDFromIndex turns a subzone's index within its parent zone into its
// SubzoneID.
func SubzoneIDFromIndex(idx int) SubzoneID {
Expand Down
42 changes: 31 additions & 11 deletions pkg/storage/reports/constraint_stats_report.go
Original file line number Diff line number Diff line change
Expand Up @@ -379,6 +379,12 @@ type constraintConformanceVisitor struct {

report *replicationConstraintStatsReportSaver
visitErr bool

// prevZoneKey and prevConstraints maintain state from one range to the next.
// This state can be reused when a range is covered by the same zone config as
// the previous one. Reusing it speeds up the report generation.
prevZoneKey ZoneKey
prevConstraints []config.Constraints
}

var _ rangeVisitor = &constraintConformanceVisitor{}
Expand Down Expand Up @@ -406,6 +412,8 @@ func (v *constraintConformanceVisitor) failed() bool {
// reset is part of the rangeVisitor interface.
func (v *constraintConformanceVisitor) reset(ctx context.Context) {
v.visitErr = false
v.prevZoneKey = ZoneKey{}
v.prevConstraints = nil
v.report.resetReport()

// Iterate through all the zone configs to create report entries for all the
Expand All @@ -428,23 +436,19 @@ func (v *constraintConformanceVisitor) reset(ctx context.Context) {
}
}

// constraintConformanceVisitor is part of the rangeVisitor interface.
func (v *constraintConformanceVisitor) visit(
ctx context.Context, r roachpb.RangeDescriptor,
// visitNewZone is part of the rangeVisitor interface.
func (v *constraintConformanceVisitor) visitNewZone(
ctx context.Context, r *roachpb.RangeDescriptor,
) (retErr error) {

defer func() {
if retErr != nil {
v.visitErr = true
}
v.visitErr = retErr != nil
}()

storeDescs := v.storeResolver(r)

// Find the applicable constraints, which may be inherited.
var constraints []config.Constraints
var zKey ZoneKey
_, err := visitZones(ctx, r, v.cfg,
_, err := visitZones(ctx, r, v.cfg, ignoreSubzonePlaceholders,
func(_ context.Context, zone *config.ZoneConfig, key ZoneKey) bool {
if zone.Constraints == nil {
return false
Expand All @@ -456,10 +460,26 @@ func (v *constraintConformanceVisitor) visit(
if err != nil {
return errors.Errorf("unexpected error visiting zones: %s", err)
}
v.prevZoneKey = zKey
v.prevConstraints = constraints
v.countRange(ctx, r, zKey, constraints)
return nil
}

// visitSameZone is part of the rangeVisitor interface.
func (v *constraintConformanceVisitor) visitSameZone(
ctx context.Context, r *roachpb.RangeDescriptor,
) (retErr error) {
v.countRange(ctx, r, v.prevZoneKey, v.prevConstraints)
return nil
}

func (v *constraintConformanceVisitor) countRange(
ctx context.Context, r *roachpb.RangeDescriptor, key ZoneKey, constraints []config.Constraints,
) {
storeDescs := v.storeResolver(r)
violated := processRange(ctx, storeDescs, constraints)
for _, c := range violated {
v.report.AddViolation(zKey, Constraint, c)
v.report.AddViolation(key, Constraint, c)
}
return nil
}
167 changes: 97 additions & 70 deletions pkg/storage/reports/constraint_stats_report_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -32,6 +32,7 @@ import (
"github.com/cockroachdb/cockroach/pkg/testutils/serverutils"
"github.com/cockroachdb/cockroach/pkg/util/encoding"
"github.com/cockroachdb/cockroach/pkg/util/hlc"
"github.com/cockroachdb/cockroach/pkg/util/keysutil"
"github.com/cockroachdb/cockroach/pkg/util/leaktest"
"github.com/cockroachdb/cockroach/pkg/util/uuid"
"github.com/cockroachdb/errors"
Expand All @@ -44,8 +45,8 @@ func TestConformanceReport(t *testing.T) {
defer leaktest.AfterTest(t)()
tests := []conformanceConstraintTestCase{
{
name: "simple no violations",
baseReportTestCase: baseReportTestCase{
name: "simple no violations",
defaultZone: zone{replicas: 3},
schema: []database{
{
Expand Down Expand Up @@ -90,9 +91,9 @@ func TestConformanceReport(t *testing.T) {
}},
},
{
name: "violations at multiple levels",
// Test zone constraints inheritance at all levels.
baseReportTestCase: baseReportTestCase{
name: "violations at multiple levels",
defaultZone: zone{replicas: 3, constraints: "[+default]"},
schema: []database{
{
Expand Down Expand Up @@ -363,6 +364,19 @@ func (t table) validate() error {
return t.partitions.validate()
}

func (t *table) addPKIdx() {
if len(t.indexes) > 0 && t.indexes[0].name == "PK" {
return
}
// Add the PK index if missing.
pkIdx := index{
name: "PK",
zone: nil,
partitions: t.partitions,
}
t.indexes = append([]index{pkIdx}, t.indexes...)
}

type database struct {
name string
tables []table
Expand Down Expand Up @@ -451,11 +465,11 @@ func (n node) toDescriptors() (roachpb.NodeDescriptor, []roachpb.StoreDescriptor

type conformanceConstraintTestCase struct {
baseReportTestCase
exp []constraintEntry
name string
exp []constraintEntry
}

type baseReportTestCase struct {
name string
schema []database
splits []split
nodes []node
Expand Down Expand Up @@ -698,79 +712,35 @@ func compileTestCase(tc baseReportTestCase) (compiledTestCase, error) {
objectCounter++
tableToID[table.name] = tableID

pkIdx := index{
name: "PK",
zone: nil,
partitions: table.partitions,
}
table.indexes = append(append([]index(nil), pkIdx), table.indexes...)

// Create a table descriptor to be used for creating the zone config.
table.addPKIdx()
tableDesc, err := makeTableDesc(table, tableID, dbID)
if err != nil {
return compiledTestCase{}, errors.Wrap(err, "error creating table descriptor")
}
sysCfgBuilder.addTableDesc(tableID, tableDesc)

// Create the table's zone config.
var tableZone *config.ZoneConfig
if table.zone != nil {
tableZone = new(config.ZoneConfig)
*tableZone = table.zone.toZoneConfig()
}
// Add subzones for the PK partitions.
tableZone = addIndexSubzones(
table.indexes[0], tableZone, tableDesc, 1 /* id of PK */)
// Add subzones for all the indexes.
for i, idx := range table.indexes {
idxID := i + 1 // index 1 is the PK
idxToID[fmt.Sprintf("%s.%s", table.name, idx.name)] = idxID
tableZone = addIndexSubzones(idx, tableZone, tableDesc, idxID)
tableZone, err := generateTableZone(table, tableDesc)
if err != nil {
return compiledTestCase{}, err
}
// Fill in the SubzoneSpans.
if tableZone != nil {
var err error
tableZone.SubzoneSpans, err = sql.GenerateSubzoneSpans(
nil, uuid.UUID{} /* clusterID */, &tableDesc, tableZone.Subzones,
false /* hasNewSubzones */)
if err != nil {
return compiledTestCase{}, errors.Wrap(err, "error generating subzone spans")
}
if err := sysCfgBuilder.addTableZone(tableDesc, tableID, *tableZone); err != nil {
if err := sysCfgBuilder.addTableZone(tableDesc, *tableZone); err != nil {
return compiledTestCase{}, err
}
}
// Add the indexes to idxToID.
for i, idx := range table.indexes {
idxID := i + 1 // index 1 is the PK
idxToID[fmt.Sprintf("%s.%s", table.name, idx.name)] = idxID
}
}
}

keyScanner := keysutils.MakePrettyScannerForNamedTables(tableToID, idxToID)
ranges := make([]roachpb.RangeDescriptor, len(tc.splits))
for i, split := range tc.splits {
prettyKey := tc.splits[i].key
startKey, err := keyScanner.Scan(split.key)
if err != nil {
return compiledTestCase{}, errors.Wrapf(err, "failed to parse key: %s", prettyKey)
}
var endKey roachpb.Key
if i < len(tc.splits)-1 {
prettyKey := tc.splits[i+1].key
endKey, err = keyScanner.Scan(prettyKey)
if err != nil {
return compiledTestCase{}, errors.Wrapf(err, "failed to parse key: %s", prettyKey)
}
} else {
endKey = roachpb.KeyMax
}

rd := roachpb.RangeDescriptor{
RangeID: roachpb.RangeID(i + 1), // IDs start at 1
StartKey: keys.MustAddr(startKey),
EndKey: keys.MustAddr(endKey),
}
for _, storeID := range split.stores {
rd.AddReplica(roachpb.NodeID(storeID), roachpb.StoreID(storeID), roachpb.VOTER_FULL)
}
ranges[i] = rd
ranges, err := processSplits(keyScanner, tc.splits)
if err != nil {
return compiledTestCase{}, err
}

var storeDescs []roachpb.StoreDescriptor
Expand All @@ -781,7 +751,7 @@ func compileTestCase(tc baseReportTestCase) (compiledTestCase, error) {
}
storeDescs = append(storeDescs, sds...)
}
storeResolver := func(r roachpb.RangeDescriptor) []roachpb.StoreDescriptor {
storeResolver := func(r *roachpb.RangeDescriptor) []roachpb.StoreDescriptor {
stores := make([]roachpb.StoreDescriptor, len(r.Replicas().Voters()))
for i, rep := range r.Replicas().Voters() {
for _, desc := range storeDescs {
Expand Down Expand Up @@ -817,6 +787,67 @@ func compileTestCase(tc baseReportTestCase) (compiledTestCase, error) {
}, nil
}

func generateTableZone(t table, tableDesc sqlbase.TableDescriptor) (*config.ZoneConfig, error) {
// Create the table's zone config.
var tableZone *config.ZoneConfig
if t.zone != nil {
tableZone = new(config.ZoneConfig)
*tableZone = t.zone.toZoneConfig()
}
// Add subzones for the PK partitions.
tableZone = addIndexSubzones(t.indexes[0], tableZone, 1 /* id of PK */)
// Add subzones for all the indexes.
for i, idx := range t.indexes {
idxID := i + 1 // index 1 is the PK
tableZone = addIndexSubzones(idx, tableZone, idxID)
}
// Fill in the SubzoneSpans.
if tableZone != nil {
var err error
tableZone.SubzoneSpans, err = sql.GenerateSubzoneSpans(
nil, uuid.UUID{} /* clusterID */, &tableDesc, tableZone.Subzones,
false /* hasNewSubzones */)
if err != nil {
return nil, errors.Wrap(err, "error generating subzone spans")
}
}
return tableZone, nil
}

func processSplits(
keyScanner keysutil.PrettyScanner, splits []split,
) ([]roachpb.RangeDescriptor, error) {
ranges := make([]roachpb.RangeDescriptor, len(splits))
for i, split := range splits {
prettyKey := splits[i].key
startKey, err := keyScanner.Scan(split.key)
if err != nil {
return nil, errors.Wrapf(err, "failed to parse key: %s", prettyKey)
}
var endKey roachpb.Key
if i < len(splits)-1 {
prettyKey := splits[i+1].key
endKey, err = keyScanner.Scan(prettyKey)
if err != nil {
return nil, errors.Wrapf(err, "failed to parse key: %s", prettyKey)
}
} else {
endKey = roachpb.KeyMax
}

rd := roachpb.RangeDescriptor{
RangeID: roachpb.RangeID(i + 1), // IDs start at 1
StartKey: keys.MustAddr(startKey),
EndKey: keys.MustAddr(endKey),
}
for _, storeID := range split.stores {
rd.AddReplica(roachpb.NodeID(storeID), roachpb.StoreID(storeID), roachpb.VOTER_FULL)
}
ranges[i] = rd
}
return ranges, nil
}

func makeTableDesc(t table, tableID int, dbID int) (sqlbase.TableDescriptor, error) {
if err := t.validate(); err != nil {
return sqlbase.TableDescriptor{}, err
Expand Down Expand Up @@ -857,9 +888,7 @@ func makeTableDesc(t table, tableID int, dbID int) (sqlbase.TableDescriptor, err
//
// parent: Can be nil if the parent table doesn't have a zone of its own. In that
// case, if any subzones are created, a placeholder zone will also be created and returned.
func addIndexSubzones(
idx index, parent *config.ZoneConfig, tableDesc sql.TableDescriptor, idxID int,
) *config.ZoneConfig {
func addIndexSubzones(idx index, parent *config.ZoneConfig, idxID int) *config.ZoneConfig {
res := parent

ensureParent := func() {
Expand Down Expand Up @@ -937,10 +966,8 @@ func (b *systemConfigBuilder) addDatabaseZone(name string, id int, cfg config.Zo
return b.addZoneInner(name, id, cfg)
}

func (b *systemConfigBuilder) addTableZone(
t sqlbase.TableDescriptor, id int, cfg config.ZoneConfig,
) error {
if err := b.addZoneInner(t.Name, id, cfg); err != nil {
func (b *systemConfigBuilder) addTableZone(t sqlbase.TableDescriptor, cfg config.ZoneConfig) error {
if err := b.addZoneInner(t.Name, int(t.ID), cfg); err != nil {
return err
}
// Figure out the mapping from all the partition names to zone keys.
Expand All @@ -959,7 +986,7 @@ func (b *systemConfigBuilder) addTableZone(
object = fmt.Sprintf("%s.%s", idx, subzone.PartitionName)
}
if err := b.addZoneToObjectMapping(
MakeZoneKey(uint32(id), base.SubzoneIDFromIndex(i)), object,
MakeZoneKey(uint32(t.ID), base.SubzoneIDFromIndex(i)), object,
); err != nil {
return err
}
Expand Down
Loading

0 comments on commit 2e21595

Please sign in to comment.