Skip to content

Commit 92548d5

Browse files
committed
infoschema: fix issue of information schema cache miss cause by schema version gap (pingcap#53445)
close pingcap#53428 Signed-off-by: crazycs520 <[email protected]>
1 parent e4f48ef commit 92548d5

File tree

4 files changed

+115
-7
lines changed

4 files changed

+115
-7
lines changed

domain/domain.go

+2
Original file line numberDiff line numberDiff line change
@@ -417,6 +417,8 @@ func (do *Domain) tryLoadSchemaDiffs(m *meta.Meta, usedVersion, newVersion int64
417417
if diff == nil {
418418
// Empty diff means the txn of generating schema version is committed, but the txn of `runDDLJob` is not or fail.
419419
// It is safe to skip the empty diff because the infoschema is new enough and consistent.
420+
logutil.BgLogger().Info("diff load InfoSchema get empty schema diff", zap.Int64("version", usedVersion))
421+
do.infoCache.InsertEmptySchemaVersion(usedVersion)
420422
continue
421423
}
422424
diffs = append(diffs, diff)

infoschema/cache.go

+65-6
Original file line numberDiff line numberDiff line change
@@ -30,6 +30,9 @@ type InfoCache struct {
3030
mu sync.RWMutex
3131
// cache is sorted by both SchemaVersion and timestamp in descending order, assume they have same order
3232
cache []schemaAndTimestamp
33+
34+
// emptySchemaVersions stores schema version which has no schema_diff.
35+
emptySchemaVersions map[int64]struct{}
3336
}
3437

3538
type schemaAndTimestamp struct {
@@ -40,10 +43,18 @@ type schemaAndTimestamp struct {
4043
// NewCache creates a new InfoCache.
4144
func NewCache(capacity int) *InfoCache {
4245
return &InfoCache{
43-
cache: make([]schemaAndTimestamp, 0, capacity),
46+
cache: make([]schemaAndTimestamp, 0, capacity),
47+
emptySchemaVersions: make(map[int64]struct{}),
4448
}
4549
}
4650

51+
// Size returns the size of the cache, export for test.
52+
func (h *InfoCache) Size() int {
53+
h.mu.Lock()
54+
defer h.mu.Unlock()
55+
return len(h.cache)
56+
}
57+
4758
// Reset resets the cache.
4859
func (h *InfoCache) Reset(capacity int) {
4960
h.mu.Lock()
@@ -63,6 +74,11 @@ func (h *InfoCache) GetLatest() InfoSchema {
6374
return nil
6475
}
6576

77+
// GetEmptySchemaVersions returns emptySchemaVersions, exports for testing.
78+
func (h *InfoCache) GetEmptySchemaVersions() map[int64]struct{} {
79+
return h.emptySchemaVersions
80+
}
81+
6682
func (h *InfoCache) getSchemaByTimestampNoLock(ts uint64) (InfoSchema, bool) {
6783
logutil.BgLogger().Debug("SCHEMA CACHE get schema", zap.Uint64("timestamp", ts))
6884
// search one by one instead of binary search, because the timestamp of a schema could be 0
@@ -80,11 +96,32 @@ func (h *InfoCache) getSchemaByTimestampNoLock(ts uint64) (InfoSchema, bool) {
8096
// the first element is the latest schema, so we can return it directly.
8197
return is.infoschema, true
8298
}
83-
if h.cache[i-1].infoschema.SchemaMetaVersion() == is.infoschema.SchemaMetaVersion()+1 && uint64(h.cache[i-1].timestamp) > ts {
84-
// This first condition is to make sure the schema version is continuous. If last(cache[i-1]) schema-version is 10,
85-
// but current(cache[i]) schema-version is not 9, then current schema is not suitable for ts.
86-
// The second condition is to make sure the cache[i-1].timestamp > ts >= cache[i].timestamp, then the current schema is suitable for ts.
87-
return is.infoschema, true
99+
100+
if uint64(h.cache[i-1].timestamp) > ts {
101+
// The first condition is to make sure the cache[i-1].timestamp > ts >= cache[i].timestamp, then the current schema is suitable for ts.
102+
lastVersion := h.cache[i-1].infoschema.SchemaMetaVersion()
103+
currentVersion := is.infoschema.SchemaMetaVersion()
104+
if lastVersion == currentVersion+1 {
105+
// This condition is to make sure the schema version is continuous. If last(cache[i-1]) schema-version is 10,
106+
// but current(cache[i]) schema-version is not 9, then current schema may not suitable for ts.
107+
return is.infoschema, true
108+
}
109+
if lastVersion > currentVersion {
110+
found := true
111+
for ver := currentVersion + 1; ver < lastVersion; ver++ {
112+
_, ok := h.emptySchemaVersions[ver]
113+
if !ok {
114+
found = false
115+
break
116+
}
117+
}
118+
if found {
119+
// This condition is to make sure the schema version is continuous. If last(cache[i-1]) schema-version is 10, and
120+
// current(cache[i]) schema-version is 8, then there is a gap exist, and if all the gap version can be found in cache.emptySchemaVersions
121+
// which means those gap versions don't have schema info, then current schema is also suitable for ts.
122+
return is.infoschema, true
123+
}
124+
}
88125
}
89126
// current schema is not suitable for ts, then break the loop to avoid the unnecessary search.
90127
break
@@ -194,3 +231,25 @@ func (h *InfoCache) Insert(is InfoSchema, schemaTS uint64) bool {
194231

195232
return true
196233
}
234+
235+
// InsertEmptySchemaVersion inserts empty schema version into a map. If exceeded the cache capacity, remove the oldest version.
236+
func (h *InfoCache) InsertEmptySchemaVersion(version int64) {
237+
h.mu.Lock()
238+
defer h.mu.Unlock()
239+
240+
h.emptySchemaVersions[version] = struct{}{}
241+
if len(h.emptySchemaVersions) > cap(h.cache) {
242+
// remove oldest version.
243+
versions := make([]int64, 0, len(h.emptySchemaVersions))
244+
for ver := range h.emptySchemaVersions {
245+
versions = append(versions, ver)
246+
}
247+
sort.Slice(versions, func(i, j int) bool { return versions[i] < versions[j] })
248+
for _, ver := range versions {
249+
delete(h.emptySchemaVersions, ver)
250+
if len(h.emptySchemaVersions) <= cap(h.cache) {
251+
break
252+
}
253+
}
254+
}
255+
}

infoschema/cache_test.go

+47
Original file line numberDiff line numberDiff line change
@@ -244,4 +244,51 @@ func TestCacheWithSchemaTsZero(t *testing.T) {
244244
checkFn(1, 84, false)
245245
checkFn(85, 100, true)
246246
require.Equal(t, 16, ic.Size())
247+
248+
// Test cache with schema version hole, which is cause by schema version doesn't has related schema-diff.
249+
ic = infoschema.NewCache(16)
250+
require.NotNil(t, ic)
251+
for i := 1; i <= 8; i++ {
252+
ic.Insert(infoschema.MockInfoSchemaWithSchemaVer(nil, int64(i)), uint64(i))
253+
}
254+
checkFn(1, 10, true)
255+
// mock for schema version hole, schema-version 9 is missing.
256+
ic.Insert(infoschema.MockInfoSchemaWithSchemaVer(nil, 10), 10)
257+
checkFn(1, 7, true)
258+
// without empty schema version map, get snapshot by ts 8, 9 will both failed.
259+
checkFn(8, 9, false)
260+
checkFn(10, 10, true)
261+
// add empty schema version 9.
262+
ic.InsertEmptySchemaVersion(9)
263+
// after set empty schema version, get snapshot by ts 8, 9 will both success.
264+
checkFn(1, 8, true)
265+
checkFn(10, 10, true)
266+
is := ic.GetBySnapshotTS(uint64(9))
267+
require.NotNil(t, is)
268+
// since schema version 9 is empty, so get by ts 9 will get schema which version is 8.
269+
require.Equal(t, int64(8), is.SchemaMetaVersion())
270+
}
271+
272+
func TestCacheEmptySchemaVersion(t *testing.T) {
273+
ic := infoschema.NewCache(16)
274+
require.NotNil(t, ic)
275+
require.Equal(t, 0, len(ic.GetEmptySchemaVersions()))
276+
for i := 0; i < 16; i++ {
277+
ic.InsertEmptySchemaVersion(int64(i))
278+
}
279+
emptyVersions := ic.GetEmptySchemaVersions()
280+
require.Equal(t, 16, len(emptyVersions))
281+
for i := 0; i < 16; i++ {
282+
_, ok := emptyVersions[int64(i)]
283+
require.True(t, ok)
284+
}
285+
for i := 16; i < 20; i++ {
286+
ic.InsertEmptySchemaVersion(int64(i))
287+
}
288+
emptyVersions = ic.GetEmptySchemaVersions()
289+
require.Equal(t, 16, len(emptyVersions))
290+
for i := 4; i < 20; i++ {
291+
_, ok := emptyVersions[int64(i)]
292+
require.True(t, ok)
293+
}
247294
}

statistics/handle/handletest/globalstats/BUILD.bazel

+1-1
Original file line numberDiff line numberDiff line change
@@ -9,7 +9,7 @@ go_test(
99
"main_test.go",
1010
],
1111
flaky = True,
12-
shard_count = 7,
12+
shard_count = 8,
1313
deps = [
1414
"//domain",
1515
"//testkit",

0 commit comments

Comments
 (0)