Skip to content

Commit 3c8f2f3

Browse files
authored
executor: fix outer join bug in hash join v2 (#56855)
close #56856
1 parent 7292117 commit 3c8f2f3

File tree

6 files changed

+122
-12
lines changed

6 files changed

+122
-12
lines changed

pkg/executor/join/inner_join_probe_test.go

Lines changed: 69 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -196,10 +196,56 @@ func checkChunksEqual(t *testing.T, expectedChunks []*chunk.Chunk, resultChunks
196196
}
197197
}
198198

199+
// copy data from src to dst, the caller should ensure that src.NumCols() >= dst.NumCols()
200+
func copySelectedRows(src *chunk.Chunk, dst *chunk.Chunk, selected []bool) (bool, error) {
201+
if src.NumRows() == 0 {
202+
return false, nil
203+
}
204+
if src.Sel() != nil || dst.Sel() != nil {
205+
return false, errors.New("copy with sel")
206+
}
207+
if src.NumCols() == 0 {
208+
numSelected := 0
209+
for _, s := range selected {
210+
if s {
211+
numSelected++
212+
}
213+
}
214+
dst.SetNumVirtualRows(dst.GetNumVirtualRows() + numSelected)
215+
return numSelected > 0, nil
216+
}
217+
218+
oldLen := dst.NumRows()
219+
for j := 0; j < src.NumCols(); j++ {
220+
if j >= dst.NumCols() {
221+
break
222+
}
223+
srcCol := src.Column(j)
224+
dstCol := dst.Column(j)
225+
chunk.CopySelectedRows(dstCol, srcCol, selected)
226+
}
227+
numSelected := dst.NumRows() - oldLen
228+
dst.SetNumVirtualRows(dst.GetNumVirtualRows() + numSelected)
229+
return numSelected > 0, nil
230+
}
231+
199232
func testJoinProbe(t *testing.T, withSel bool, leftKeyIndex []int, rightKeyIndex []int, leftKeyTypes []*types.FieldType, rightKeyTypes []*types.FieldType,
200233
leftTypes []*types.FieldType, rightTypes []*types.FieldType, rightAsBuildSide bool, leftUsed []int, rightUsed []int,
201234
leftUsedByOtherCondition []int, rightUsedByOtherCondition []int, leftFilter expression.CNFExprs, rightFilter expression.CNFExprs,
202235
otherCondition expression.CNFExprs, partitionNumber int, joinType logicalop.JoinType, inputRowNumber int) {
236+
// leftUsed/rightUsed is nil, it means select all columns
237+
if leftUsed == nil {
238+
leftUsed = make([]int, 0)
239+
for index := range leftTypes {
240+
leftUsed = append(leftUsed, index)
241+
}
242+
}
243+
if rightUsed == nil {
244+
rightUsed = make([]int, 0)
245+
for index := range rightTypes {
246+
rightUsed = append(rightUsed, index)
247+
}
248+
}
203249
buildKeyIndex, probeKeyIndex := leftKeyIndex, rightKeyIndex
204250
buildKeyTypes, probeKeyTypes := leftKeyTypes, rightKeyTypes
205251
buildTypes, probeTypes := leftTypes, rightTypes
@@ -305,18 +351,27 @@ func testJoinProbe(t *testing.T, withSel bool, leftKeyIndex []int, rightKeyIndex
305351
}
306352
}
307353
// check if build column can be inserted to probe column directly
308-
for i := 0; i < len(buildTypes); i++ {
354+
for i := 0; i < min(len(buildTypes), len(probeTypes)); i++ {
309355
buildLength := chunk.GetFixedLen(buildTypes[i])
310356
probeLength := chunk.GetFixedLen(probeTypes[i])
311357
require.Equal(t, buildLength, probeLength, "build type and probe type is not compatible")
312358
}
313359
for i := 0; i < chunkNumber; i++ {
314-
buildChunks = append(buildChunks, testutil.GenRandomChunks(buildTypes, inputRowNumber))
315-
probeChunk := testutil.GenRandomChunks(probeTypes, inputRowNumber*2/3)
316-
// copy some build data to probe side, to make sure there is some matched rows
317-
_, err := chunk.CopySelectedJoinRowsDirect(buildChunks[i], selected, probeChunk)
318-
probeChunks = append(probeChunks, probeChunk)
319-
require.NoError(t, err)
360+
if len(buildTypes) >= len(probeTypes) {
361+
buildChunks = append(buildChunks, testutil.GenRandomChunks(buildTypes, inputRowNumber))
362+
probeChunk := testutil.GenRandomChunks(probeTypes, inputRowNumber*2/3)
363+
// copy some build data to probe side, to make sure there is some matched rows
364+
_, err := copySelectedRows(buildChunks[i], probeChunk, selected)
365+
require.NoError(t, err)
366+
probeChunks = append(probeChunks, probeChunk)
367+
} else {
368+
probeChunks = append(probeChunks, testutil.GenRandomChunks(probeTypes, inputRowNumber))
369+
buildChunk := testutil.GenRandomChunks(buildTypes, inputRowNumber*2/3)
370+
// copy some build data to probe side, to make sure there is some matched rows
371+
_, err := copySelectedRows(probeChunks[i], buildChunk, selected)
372+
require.NoError(t, err)
373+
buildChunks = append(buildChunks, buildChunk)
374+
}
320375
}
321376

322377
if withSel {
@@ -436,14 +491,16 @@ func TestInnerJoinProbeBasic(t *testing.T) {
436491

437492
lTypes := []*types.FieldType{intTp, stringTp, uintTp, stringTp, tinyTp}
438493
rTypes := []*types.FieldType{intTp, stringTp, uintTp, stringTp, tinyTp}
494+
rTypes = append(rTypes, retTypes...)
439495
rTypes1 := []*types.FieldType{uintTp, stringTp, intTp, stringTp, tinyTp}
496+
rTypes1 = append(rTypes1, rTypes1...)
440497

441498
rightAsBuildSide := []bool{true, false}
442499
partitionNumber := 4
443500

444501
testCases := []testCase{
445502
// normal case
446-
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, []int{0, 1, 2, 3}, []int{0, 1, 2, 3}, nil, nil, nil},
503+
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, nil, nil, nil, nil, nil},
447504
// rightUsed is empty
448505
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, []int{0, 1, 2, 3}, []int{}, nil, nil, nil},
449506
// leftUsed is empty
@@ -570,6 +627,7 @@ func TestInnerJoinProbeOtherCondition(t *testing.T) {
570627

571628
lTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
572629
rTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
630+
rTypes = append(rTypes, rTypes...)
573631

574632
tinyTp := types.NewFieldType(mysql.TypeTiny)
575633
a := &expression.Column{Index: 1, RetType: nullableIntTp}
@@ -585,6 +643,7 @@ func TestInnerJoinProbeOtherCondition(t *testing.T) {
585643
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightAsBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, nil, otherCondition, partitionNumber, logicalop.InnerJoin, 200)
586644
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightAsBuild, []int{}, []int{}, []int{1}, []int{3}, nil, nil, otherCondition, partitionNumber, logicalop.InnerJoin, 200)
587645
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightAsBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, nil, otherCondition, partitionNumber, logicalop.InnerJoin, 200)
646+
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightAsBuild, nil, nil, []int{1}, []int{3}, nil, nil, otherCondition, partitionNumber, logicalop.InnerJoin, 200)
588647
}
589648
}
590649

@@ -602,6 +661,7 @@ func TestInnerJoinProbeWithSel(t *testing.T) {
602661

603662
lTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
604663
rTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
664+
rTypes = append(rTypes, rTypes...)
605665

606666
tinyTp := types.NewFieldType(mysql.TypeTiny)
607667
a := &expression.Column{Index: 1, RetType: nullableIntTp}
@@ -620,6 +680,7 @@ func TestInnerJoinProbeWithSel(t *testing.T) {
620680
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightAsBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, nil, oc, partitionNumber, logicalop.InnerJoin, 500)
621681
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightAsBuild, []int{}, []int{}, []int{1}, []int{3}, nil, nil, oc, partitionNumber, logicalop.InnerJoin, 500)
622682
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightAsBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, nil, oc, partitionNumber, logicalop.InnerJoin, 500)
683+
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightAsBuild, nil, nil, []int{1}, []int{3}, nil, nil, oc, partitionNumber, logicalop.InnerJoin, 500)
623684
}
624685
}
625686
}

pkg/executor/join/left_outer_join_probe_test.go

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -120,7 +120,9 @@ func TestLeftOuterJoinProbeBasic(t *testing.T) {
120120

121121
lTypes := []*types.FieldType{intTp, stringTp, uintTp, stringTp, tinyTp}
122122
rTypes := []*types.FieldType{intTp, stringTp, uintTp, stringTp, tinyTp}
123+
rTypes = append(rTypes, retTypes...)
123124
rTypes1 := []*types.FieldType{uintTp, stringTp, intTp, stringTp, tinyTp}
125+
rTypes1 = append(rTypes1, rTypes1...)
124126

125127
rightAsBuildSide := []bool{true, false}
126128
partitionNumber := 4
@@ -129,7 +131,7 @@ func TestLeftOuterJoinProbeBasic(t *testing.T) {
129131

130132
testCases := []testCase{
131133
// normal case
132-
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, []int{0, 1, 2, 3}, []int{0, 1, 2, 3}, nil, nil, nil},
134+
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, nil, nil, nil, nil, nil},
133135
// rightUsed is empty
134136
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, []int{0, 1, 2, 3}, []int{}, nil, nil, nil},
135137
// leftUsed is empty
@@ -262,6 +264,7 @@ func TestLeftOuterJoinProbeOtherCondition(t *testing.T) {
262264

263265
lTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
264266
rTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
267+
rTypes = append(rTypes, rTypes...)
265268

266269
tinyTp := types.NewFieldType(mysql.TypeTiny)
267270
a := &expression.Column{Index: 1, RetType: nullableIntTp}
@@ -285,6 +288,7 @@ func TestLeftOuterJoinProbeOtherCondition(t *testing.T) {
285288
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 200)
286289
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{}, []int{}, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 200)
287290
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 200)
291+
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, nil, nil, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 200)
288292
}
289293
}
290294
}
@@ -303,6 +307,7 @@ func TestLeftOuterJoinProbeWithSel(t *testing.T) {
303307

304308
lTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
305309
rTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
310+
rTypes = append(rTypes, rTypes...)
306311

307312
tinyTp := types.NewFieldType(mysql.TypeTiny)
308313
a := &expression.Column{Index: 1, RetType: nullableIntTp}
@@ -326,6 +331,7 @@ func TestLeftOuterJoinProbeWithSel(t *testing.T) {
326331
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 500)
327332
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{}, []int{}, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 500)
328333
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 500)
334+
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, nil, nil, []int{1}, []int{3}, leftFilter, nil, otherCondition, partitionNumber, joinType, 500)
329335
}
330336
}
331337
}

pkg/executor/join/outer_join_probe.go

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -165,7 +165,7 @@ func (j *outerJoinProbe) ScanRowTable(joinResult *hashjoinWorkerResult, sqlKille
165165
}
166166

167167
func (j *outerJoinProbe) buildResultForMatchedRowsAfterOtherCondition(chk, joinedChk *chunk.Chunk) {
168-
probeColOffsetInJoinedChunk, buildColOffsetInJoinedChunk := j.currentChunk.NumCols(), 0
168+
probeColOffsetInJoinedChunk, buildColOffsetInJoinedChunk := j.ctx.hashTableMeta.totalColumnNumber, 0
169169
if j.rightAsBuildSide {
170170
probeColOffsetInJoinedChunk, buildColOffsetInJoinedChunk = 0, j.currentChunk.NumCols()
171171
}

pkg/executor/join/right_outer_join_probe_test.go

Lines changed: 7 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -120,7 +120,9 @@ func TestRightOuterJoinProbeBasic(t *testing.T) {
120120

121121
lTypes := []*types.FieldType{intTp, stringTp, uintTp, stringTp, tinyTp}
122122
rTypes := []*types.FieldType{intTp, stringTp, uintTp, stringTp, tinyTp}
123+
rTypes = append(rTypes, retTypes...)
123124
rTypes1 := []*types.FieldType{uintTp, stringTp, intTp, stringTp, tinyTp}
125+
rTypes1 = append(rTypes1, rTypes1...)
124126

125127
rightAsBuildSide := []bool{true, false}
126128
partitionNumber := 3
@@ -129,7 +131,7 @@ func TestRightOuterJoinProbeBasic(t *testing.T) {
129131

130132
testCases := []testCase{
131133
// normal case
132-
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, []int{0, 1, 2, 3}, []int{0, 1, 2, 3}, nil, nil, nil},
134+
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, nil, nil, nil, nil, nil},
133135
// rightUsed is empty
134136
{[]int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, []int{0, 1, 2, 3}, []int{}, nil, nil, nil},
135137
// leftUsed is empty
@@ -261,6 +263,7 @@ func TestRightOuterJoinProbeOtherCondition(t *testing.T) {
261263

262264
lTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
263265
rTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
266+
rTypes = append(rTypes, retTypes...)
264267

265268
tinyTp := types.NewFieldType(mysql.TypeTiny)
266269
a := &expression.Column{Index: 1, RetType: nullableIntTp}
@@ -283,6 +286,7 @@ func TestRightOuterJoinProbeOtherCondition(t *testing.T) {
283286
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 200)
284287
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{}, []int{}, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 200)
285288
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 200)
289+
testJoinProbe(t, false, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, nil, nil, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 200)
286290
}
287291
}
288292
}
@@ -301,6 +305,7 @@ func TestRightOuterJoinProbeWithSel(t *testing.T) {
301305

302306
lTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
303307
rTypes := []*types.FieldType{intTp, intTp, stringTp, uintTp, stringTp}
308+
rTypes = append(rTypes, retTypes...)
304309

305310
tinyTp := types.NewFieldType(mysql.TypeTiny)
306311
a := &expression.Column{Index: 1, RetType: nullableIntTp}
@@ -323,6 +328,7 @@ func TestRightOuterJoinProbeWithSel(t *testing.T) {
323328
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 500)
324329
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{intTp}, []*types.FieldType{intTp}, lTypes, rTypes, rightBuild, []int{}, []int{}, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 500)
325330
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, []int{1, 2, 4}, []int{0}, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 500)
331+
testJoinProbe(t, true, []int{0}, []int{0}, []*types.FieldType{nullableIntTp}, []*types.FieldType{nullableIntTp}, toNullableTypes(lTypes), toNullableTypes(rTypes), rightBuild, nil, nil, []int{1}, []int{3}, nil, rightFilter, otherCondition, 3, joinType, 500)
326332
}
327333
}
328334
}

pkg/executor/test/jointest/hashjoin/BUILD.bazel

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -9,7 +9,7 @@ go_test(
99
],
1010
flaky = True,
1111
race = "on",
12-
shard_count = 21,
12+
shard_count = 22,
1313
deps = [
1414
"//pkg/config",
1515
"//pkg/executor/join",

pkg/executor/test/jointest/hashjoin/hash_join_test.go

Lines changed: 37 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -679,3 +679,40 @@ func TestIssue56214(t *testing.T) {
679679
tk.MustQuery("select value, (select t1.id from t1 join t2 on t1.id = t2.id and t1.value < t2.value - t3.value + 3) d from t3 order by value").Check(testkit.Rows("10 1", "20 <nil>"))
680680
}
681681
}
682+
683+
func TestIssue56825(t *testing.T) {
684+
store := testkit.CreateMockStore(t)
685+
tk := testkit.NewTestKit(t, store)
686+
tk.MustExec("use test")
687+
tk.MustExec("drop table if exists t1;")
688+
tk.MustExec("drop table if exists t2;")
689+
tk.MustExec("create table t1(id int, col1 int)")
690+
tk.MustExec("create table t2(id int, col1 int, col2 int, col3 int, col4 int, col5 int)")
691+
tk.MustExec("insert into t1 values(1,2),(2,3)")
692+
tk.MustExec("insert into t2 values(1,2,3,4,5,6),(3,4,5,6,7,8),(4,5,6,7,8,9)")
693+
tk.MustExec("analyze table t1")
694+
tk.MustExec("analyze table t2")
695+
// t1 as build side
696+
for _, hashJoinV2 := range join.HashJoinV2Strings {
697+
tk.MustExec(hashJoinV2)
698+
tk.MustQuery("select * from t1 left join t2 on t1.id = t2.id and t1.col1 <= t2.col1 order by t1.id").Check(testkit.Rows("1 2 1 2 3 4 5 6", "2 3 <nil> <nil> <nil> <nil> <nil> <nil>"))
699+
tk.MustQuery("select * from t1 right join t2 on t1.id = t2.id and t1.col1 <= t2.col1 order by t2.id").Check(testkit.Rows("1 2 1 2 3 4 5 6", "<nil> <nil> 3 4 5 6 7 8", "<nil> <nil> 4 5 6 7 8 9"))
700+
}
701+
tk.MustExec("insert into t1 values(10,20),(11,21),(12,22),(13,23),(14,24),(15,25)")
702+
tk.MustExec("analyze table t1")
703+
// t2 as build side
704+
for _, hashJoinV2 := range join.HashJoinV2Strings {
705+
tk.MustExec(hashJoinV2)
706+
tk.MustQuery("select * from t1 left join t2 on t1.id = t2.id and t1.col1 <= t2.col1 order by t1.id").Check(testkit.Rows(
707+
"1 2 1 2 3 4 5 6",
708+
"2 3 <nil> <nil> <nil> <nil> <nil> <nil>",
709+
"10 20 <nil> <nil> <nil> <nil> <nil> <nil>",
710+
"11 21 <nil> <nil> <nil> <nil> <nil> <nil>",
711+
"12 22 <nil> <nil> <nil> <nil> <nil> <nil>",
712+
"13 23 <nil> <nil> <nil> <nil> <nil> <nil>",
713+
"14 24 <nil> <nil> <nil> <nil> <nil> <nil>",
714+
"15 25 <nil> <nil> <nil> <nil> <nil> <nil>",
715+
))
716+
tk.MustQuery("select * from t1 right join t2 on t1.id = t2.id and t1.col1 <= t2.col1 order by t2.id").Check(testkit.Rows("1 2 1 2 3 4 5 6", "<nil> <nil> 3 4 5 6 7 8", "<nil> <nil> 4 5 6 7 8 9"))
717+
}
718+
}

0 commit comments

Comments
 (0)