Skip to content

Commit

Permalink
Merge #30143
Browse files Browse the repository at this point in the history
30143: config/storage: cache zone config values to avoid repetitive deserialization r=spencerkimball a=spencerkimball

This change allows a `SystemConfig` to cache object ID -> `ZoneConfig`
lookups to avoid repetitively deserializing table descriptors and zone
configs for every replica in the system. Replicas themselves now also
cache zone config entries to avoid maintaining per-replica min/max bytes
and for making future use of additional zone config parameters simpler
(e.g. zone config GC policies, number of replicas, etc.).

Fixes #21702

Co-authored-by: Spencer Kimball <spencer.kimball@gmail.com>
  • Loading branch information
craig[bot] and spencerkimball committed Sep 19, 2018
2 parents 9a4cc88 + b90063f commit cb4dd95
Show file tree
Hide file tree
Showing 65 changed files with 498 additions and 443 deletions.
2 changes: 1 addition & 1 deletion pkg/cli/debug.go
Original file line number Diff line number Diff line change
Expand Up @@ -897,7 +897,7 @@ func parseGossipValues(gossipInfo *gossip.InfoStatus) (string, error) {
output = append(output, fmt.Sprintf("%q: %v", key, clusterID))
} else if key == gossip.KeySystemConfig {
if debugCtx.printSystemConfig {
var config config.SystemConfig
var config config.SystemConfigEntries
if err := protoutil.Unmarshal(bytes, &config); err != nil {
return "", errors.Wrapf(err, "failed to parse value for key %q", key)
}
Expand Down
84 changes: 65 additions & 19 deletions pkg/config/system.go
Original file line number Diff line number Diff line change
Expand Up @@ -23,11 +23,12 @@ import (
"github.com/cockroachdb/cockroach/pkg/keys"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/util/log"
"github.com/cockroachdb/cockroach/pkg/util/syncutil"
)

type zoneConfigHook func(
sysCfg SystemConfig, objectID uint32, keySuffix []byte,
) (zoneCfg ZoneConfig, found bool, err error)
sysCfg *SystemConfig, objectID uint32,
) (zone *ZoneConfig, placeholder *ZoneConfig, cache bool, err error)

var (
// ZoneConfigHook is a function used to lookup a zone config given a table
Expand All @@ -40,10 +41,34 @@ var (
testingLargestIDHook func(uint32) uint32
)

type zoneEntry struct {
zone *ZoneConfig
placeholder *ZoneConfig
}

// SystemConfig embeds a SystemConfigEntries message which contains an
// entry for every system descriptor (e.g. databases, tables, zone
// configs). It also has a map from object ID to unmarshaled zone
// config for caching.
type SystemConfig struct {
SystemConfigEntries
mu struct {
syncutil.RWMutex
cache map[uint32]zoneEntry
}
}

// NewSystemConfig returns an initialized instance of SystemConfig.
func NewSystemConfig() *SystemConfig {
sc := &SystemConfig{}
sc.mu.cache = map[uint32]zoneEntry{}
return sc
}

// Equal checks for equality.
//
// It assumes that s.Values and other.Values are sorted in key order.
func (s SystemConfig) Equal(other SystemConfig) bool {
func (s *SystemConfig) Equal(other *SystemConfigEntries) bool {
if len(s.Values) != len(other.Values) {
return false
}
Expand All @@ -65,7 +90,7 @@ func (s SystemConfig) Equal(other SystemConfig) bool {

// GetValue searches the kv list for 'key' and returns its
// roachpb.Value if found.
func (s SystemConfig) GetValue(key roachpb.Key) *roachpb.Value {
func (s *SystemConfig) GetValue(key roachpb.Key) *roachpb.Value {
if kv := s.get(key); kv != nil {
return &kv.Value
}
Expand All @@ -74,7 +99,7 @@ func (s SystemConfig) GetValue(key roachpb.Key) *roachpb.Value {

// get searches the kv list for 'key' and returns its roachpb.KeyValue
// if found.
func (s SystemConfig) get(key roachpb.Key) *roachpb.KeyValue {
func (s *SystemConfig) get(key roachpb.Key) *roachpb.KeyValue {
if index, found := s.GetIndex(key); found {
// TODO(marc): I'm pretty sure a Value returned by MVCCScan can
// never be nil. Should check.
Expand All @@ -84,7 +109,7 @@ func (s SystemConfig) get(key roachpb.Key) *roachpb.KeyValue {
}

// GetIndex searches the kv list for 'key' and returns its index if found.
func (s SystemConfig) GetIndex(key roachpb.Key) (int, bool) {
func (s *SystemConfig) GetIndex(key roachpb.Key) (int, bool) {
l := len(s.Values)
index := sort.Search(l, func(i int) bool {
return bytes.Compare(s.Values[i].Key, key) >= 0
Expand All @@ -98,7 +123,7 @@ func (s SystemConfig) GetIndex(key roachpb.Key) (int, bool) {
// GetLargestObjectID returns the largest object ID found in the config which is
// less than or equal to maxID. If maxID is 0, returns the largest ID in the
// config.
func (s SystemConfig) GetLargestObjectID(maxID uint32) (uint32, error) {
func (s *SystemConfig) GetLargestObjectID(maxID uint32) (uint32, error) {
testingLock.Lock()
hook := testingLargestIDHook
testingLock.Unlock()
Expand Down Expand Up @@ -171,9 +196,10 @@ func (s SystemConfig) GetLargestObjectID(maxID uint32) (uint32, error) {
return uint32(id), nil
}

// GetZoneConfigForKey looks up the zone config for the range containing 'key'.
// It is the caller's responsibility to ensure that the range does not need to be split.
func (s SystemConfig) GetZoneConfigForKey(key roachpb.RKey) (ZoneConfig, error) {
// GetZoneConfigForKey looks up the zone config for the object (table
// or database, specified by key.id). It is the caller's
// responsibility to ensure that the range does not need to be split.
func (s *SystemConfig) GetZoneConfigForKey(key roachpb.RKey) (*ZoneConfig, error) {
objectID, keySuffix, ok := DecodeObjectID(key)
if !ok {
// Not in the structured data namespace.
Expand All @@ -199,19 +225,39 @@ func (s SystemConfig) GetZoneConfigForKey(key roachpb.RKey) (ZoneConfig, error)
}
}

return s.getZoneConfigForID(objectID, keySuffix)
return s.getZoneConfigForKey(objectID, keySuffix)
}

// getZoneConfigForID looks up the zone config for the object (table or database)
// with 'id'.
func (s SystemConfig) getZoneConfigForID(id uint32, keySuffix []byte) (ZoneConfig, error) {
func (s *SystemConfig) getZoneConfigForKey(id uint32, keySuffix []byte) (*ZoneConfig, error) {
testingLock.Lock()
hook := ZoneConfigHook
testingLock.Unlock()
if cfg, found, err := hook(s, id, keySuffix); err != nil || found {
return cfg, err
s.mu.RLock()
entry, ok := s.mu.cache[id]
s.mu.RUnlock()
if !ok {
if zone, placeholder, cache, err := hook(s, id); err != nil {
return nil, err
} else if zone != nil {
entry = zoneEntry{zone: zone, placeholder: placeholder}
if cache {
s.mu.Lock()
s.mu.cache[id] = entry
s.mu.Unlock()
}
}
}
if entry.zone != nil {
if entry.placeholder != nil {
if subzone := entry.placeholder.GetSubzoneForKeySuffix(keySuffix); subzone != nil {
return &subzone.Config, nil
}
} else if subzone := entry.zone.GetSubzoneForKeySuffix(keySuffix); subzone != nil {
return &subzone.Config, nil
}
return entry.zone, nil
}
return DefaultZoneConfig(), nil
return DefaultZoneConfigRef(), nil
}

var staticSplits = []roachpb.RKey{
Expand Down Expand Up @@ -243,7 +289,7 @@ func StaticSplits() []roachpb.RKey {
// system ranges that come before the system tables. The system-config range is
// somewhat special in that it can contain multiple SQL tables
// (/table/0-/table/<max-system-config-desc>) within a single range.
func (s SystemConfig) ComputeSplitKey(startKey, endKey roachpb.RKey) roachpb.RKey {
func (s *SystemConfig) ComputeSplitKey(startKey, endKey roachpb.RKey) roachpb.RKey {
// Before dealing with splits necessitated by SQL tables, handle all of the
// static splits earlier in the keyspace. Note that this list must be kept in
// the proper order (ascending in the keyspace) for the logic below to work.
Expand Down Expand Up @@ -342,6 +388,6 @@ func (s SystemConfig) ComputeSplitKey(startKey, endKey roachpb.RKey) roachpb.RKe

// NeedsSplit returns whether the range [startKey, endKey) needs a split due
// to zone configs.
func (s SystemConfig) NeedsSplit(startKey, endKey roachpb.RKey) bool {
func (s *SystemConfig) NeedsSplit(startKey, endKey roachpb.RKey) bool {
return len(s.ComputeSplitKey(startKey, endKey)) > 0
}
42 changes: 21 additions & 21 deletions pkg/config/system.pb.go

Some generated files are not rendered by default. Learn more about how customized files appear on GitHub.

2 changes: 1 addition & 1 deletion pkg/config/system.proto
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,6 @@ option go_package = "config";
import "gogoproto/gogo.proto";
import "roachpb/data.proto";

message SystemConfig {
message SystemConfigEntries {
repeated roachpb.KeyValue values = 1 [(gogoproto.nullable) = false];
}
14 changes: 10 additions & 4 deletions pkg/config/system_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -256,7 +256,9 @@ func TestComputeSplitKeySystemRanges(t *testing.T) {
}

cfg := config.SystemConfig{
Values: sqlbase.MakeMetadataSchema().GetInitialValues(),
SystemConfigEntries: config.SystemConfigEntries{
Values: sqlbase.MakeMetadataSchema().GetInitialValues(),
},
}
for tcNum, tc := range testCases {
splitKey := cfg.ComputeSplitKey(tc.start, tc.end)
Expand Down Expand Up @@ -411,13 +413,17 @@ func TestGetZoneConfigForKey(t *testing.T) {
config.ZoneConfigHook = originalZoneConfigHook
}()
cfg := config.SystemConfig{
Values: sqlbase.MakeMetadataSchema().GetInitialValues(),
SystemConfigEntries: config.SystemConfigEntries{
Values: sqlbase.MakeMetadataSchema().GetInitialValues(),
},
}
for tcNum, tc := range testCases {
var objectID uint32
config.ZoneConfigHook = func(_ config.SystemConfig, id uint32, _ []byte) (config.ZoneConfig, bool, error) {
config.ZoneConfigHook = func(
_ *config.SystemConfig, id uint32,
) (*config.ZoneConfig, *config.ZoneConfig, bool, error) {
objectID = id
return config.ZoneConfig{}, false, nil
return &config.ZoneConfig{}, nil, false, nil
}
_, err := cfg.GetZoneConfigForKey(tc.key)
if err != nil {
Expand Down
6 changes: 3 additions & 3 deletions pkg/config/testutil.go
Original file line number Diff line number Diff line change
Expand Up @@ -79,11 +79,11 @@ func TestingSetZoneConfig(id uint32, zone ZoneConfig) {
testingZoneConfig[id] = zone
}

func testingZoneConfigHook(_ SystemConfig, id uint32, _ []byte) (ZoneConfig, bool, error) {
func testingZoneConfigHook(_ *SystemConfig, id uint32) (*ZoneConfig, *ZoneConfig, bool, error) {
testingLock.Lock()
defer testingLock.Unlock()
if zone, ok := testingZoneConfig[id]; ok {
return zone, true, nil
return &zone, nil, false, nil
}
return ZoneConfig{}, false, nil
return nil, nil, false, nil
}
7 changes: 7 additions & 0 deletions pkg/config/zone.go
Original file line number Diff line number Diff line change
Expand Up @@ -299,6 +299,13 @@ func DefaultZoneConfig() ZoneConfig {
return defaultZoneConfig
}

// DefaultZoneConfigRef returns a reference to the default zone config.
func DefaultZoneConfigRef() *ZoneConfig {
testingLock.Lock()
defer testingLock.Unlock()
return &defaultZoneConfig
}

// DefaultSystemZoneConfig is the default zone configuration used when no custom
// config has been specified.
func DefaultSystemZoneConfig() ZoneConfig {
Expand Down
4 changes: 2 additions & 2 deletions pkg/config/zone_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -246,8 +246,8 @@ func TestZoneConfigSubzones(t *testing.T) {

zone := DefaultZoneConfig()
subzoneAInvalid := Subzone{IndexID: 1, PartitionName: "a", Config: ZoneConfig{}}
subzoneA := Subzone{IndexID: 1, PartitionName: "a", Config: DefaultZoneConfig()}
subzoneB := Subzone{IndexID: 1, PartitionName: "b", Config: DefaultZoneConfig()}
subzoneA := Subzone{IndexID: 1, PartitionName: "a", Config: zone}
subzoneB := Subzone{IndexID: 1, PartitionName: "b", Config: zone}

if zone.IsSubzonePlaceholder() {
t.Errorf("default zone config should not be considered a subzone placeholder")
Expand Down
Loading

0 comments on commit cb4dd95

Please sign in to comment.