You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@cassandra.apache.org by "David Capwell (Jira)" <ji...@apache.org> on 2021/04/12 23:59:00 UTC

[jira] [Comment Edited] (CASSANDRA-16592) The token function in where clause return incorrect data when using token equal condition and Specified a non-exist token value

    [ https://issues.apache.org/jira/browse/CASSANDRA-16592?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17319632#comment-17319632 ] 

David Capwell edited comment on CASSANDRA-16592 at 4/12/21, 11:58 PM:
----------------------------------------------------------------------

Replicated the issue in 3.0 as well.

Here is a test to show the issue

{code}
package org.apache.cassandra.distributed.test;

import java.io.IOException;
import java.nio.ByteBuffer;

import org.junit.Test;

import org.apache.cassandra.dht.Murmur3Partitioner;
import org.apache.cassandra.distributed.Cluster;
import org.apache.cassandra.distributed.api.IInvokableInstance;
import org.apache.cassandra.distributed.api.QueryResults;
import org.apache.cassandra.distributed.shared.AssertUtils;

public class CASSANDRA16592Test extends TestBaseImpl
{
    @Test
    public void notComplex() throws IOException
    {
        try (Cluster cluster = init(Cluster.build(1).start()))
        {
            cluster.schemaChange(withKeyspace("CREATE TABLE %s.tbl (\n" +
                                              "    pk1 blob,\n" +
                                              "    ck1 text,\n" +
                                              "    ck2 text,\n" +
                                              "    t1 int,\n" +
                                              "    PRIMARY KEY ((pk1), ck1, ck2)\n" +
                                              ") WITH CLUSTERING ORDER BY (ck1 ASC, ck2 ASC)"));

            IInvokableInstance node = cluster.get(1);

            ByteBuffer targetKey = Murmur3Partitioner.LongToken.keyForToken(new Murmur3Partitioner.LongToken(-9223372036854775808L));
            node.executeInternal(withKeyspace("INSERT INTO %s.tbl(pk1, ck1, ck2, t1) VALUES (?, ?, ?, ?)"),
                                 targetKey, "match", "match", 0);

            node.executeInternal(withKeyspace("INSERT INTO %s.tbl(pk1, ck1, ck2, t1) VALUES (?, ?, ?, ?)"),
                                 Murmur3Partitioner.LongToken.keyForToken(new Murmur3Partitioner.LongToken(-9222849988925915479L)), "not match", "not match", 10);

            AssertUtils.assertRows(node.executeInternalWithResult(withKeyspace("SELECT * FROM %s.tbl WHERE token(pk1) = -9223372036854775808")),
                                   QueryResults.builder()
                                               .columns("pk1", "ck1", "ck2", "t1")
                                               .row(targetKey, "match", "match", 0)
                                               .build());
        }
    }
}
{code}


was (Author: dcapwell):
Replicated the issue in 3.0 as well.

> The token function in where clause return incorrect data when using token equal condition and Specified a non-exist token value
> -------------------------------------------------------------------------------------------------------------------------------
>
>                 Key: CASSANDRA-16592
>                 URL: https://issues.apache.org/jira/browse/CASSANDRA-16592
>             Project: Cassandra
>          Issue Type: Bug
>          Components: Legacy/CQL
>            Reporter: cimon
>            Assignee: cimon
>            Priority: Normal
>             Fix For: 3.0.x, 3.11.x, 4.0-rc
>
>
> I get incorrect value when use query like 'select Token(pk1,pk2),pk1,pk2 from ks.table1 where token(pk1,pk2) = tokenValue'. The returned token value mismatch the where condition.
> This problem is reproduced in 3.11.3 and 4.0.
> Here is my schema and select statement
> {code:java}
> // schema
> cqlsh> desc testprefix.cprefix_03 ;CREATE TABLE testprefix.cprefix_03 (
>     pk1 int,
>     pk2 int,
>     ck1 text,
>     ck2 text,
>     t1 int,
>     PRIMARY KEY ((pk1, pk2), ck1, ck2)
> ) WITH CLUSTERING ORDER BY (ck1 ASC, ck2 ASC)
>     AND additional_write_policy = '99p'
>     AND bloom_filter_fp_chance = 0.01
>     AND caching = {'keys': 'ALL', 'rows_per_partition': 'NONE'}
>     AND cdc = false
>     AND comment = ''
>     AND compaction = {'class': 'org.apache.cassandra.db.compaction.SizeTieredCompactionStrategy', 'max_threshold': '32', 'min_threshold': '4'}
>     AND compression = {'chunk_length_in_kb': '16', 'class': 'org.apache.cassandra.io.compress.LZ4Compressor'}
>     AND crc_check_chance = 1.0
>     AND default_time_to_live = 0
>     AND extensions = {}
>     AND gc_grace_seconds = 864000
>     AND max_index_interval = 2048
>     AND memtable_flush_period_in_ms = 0
>     AND min_index_interval = 128
>     AND read_repair = 'BLOCKING'
>     AND speculative_retry = '99p';
> {code}
> execute cql query
> {code:java}
> // code placeholder
> cqlsh> SELECT Token(pk1,pk2), pk1,pk2  from testprefix.cprefix_03 WHERE  token(pk1, pk2) =-9223372036854775808 LIMIT 2; 
> system.token(pk1, pk2) | pk1    | pk2
> ------------------------+--------+---------
>    -9222849988925915479 | 394560 | 3394560
>    -9222849988925915479 | 394560 | 3394560
> (2 rows)
> cqlsh> SELECT Token(pk1,pk2) from testprefix.cprefix_03 where pk1 = 394560 and pk2 = 3394560 LIMIT 2; 
> system.token(pk1, pk2)
> ------------------------
>    -9222849988925915479
>    -9222849988925915479
> (2 rows)
> cqlsh> SELECT Token(pk1,pk2), pk1,pk2  from testprefix.cprefix_03 WHERE  token(pk1, pk2) =-9222849988925915479 LIMIT 2; 
> system.token(pk1, pk2) | pk1    | pk2
> ------------------------+--------+---------
>    -9222849988925915479 | 394560 | 3394560
>    -9222849988925915479 | 394560 | 3394560
> (2 rows){code}
> we can find  that token value in the condition  are inconsistent with the values in the result.
> --------------------------------------------------------------------------------------------
> Then review the source code, to seek the anwser. 
> {code:java}
> // code placeholder
> private static void addRange(SSTableReader sstable, AbstractBounds<PartitionPosition> requested, List<AbstractBounds<PartitionPosition>> boundsList)
> {
>     if (requested instanceof Range && ((Range)requested).isWrapAround())    //  first condition
>     {
>         if (requested.right.compareTo(sstable.first) >= 0)
>         {
>             // since we wrap, we must contain the whole sstable prior to stopKey()
>             Boundary<PartitionPosition> left = new Boundary<PartitionPosition>(sstable.first, true);
>             Boundary<PartitionPosition> right;
>             right = requested.rightBoundary();
>             right = minRight(right, sstable.last, true);
>             if (!isEmpty(left, right))
>                 boundsList.add(AbstractBounds.bounds(left, right));
>         }
>         if (requested.left.compareTo(sstable.last) <= 0)
>         {
>             // since we wrap, we must contain the whole sstable after dataRange.startKey()
>             Boundary<PartitionPosition> right = new Boundary<PartitionPosition>(sstable.last, true);
>             Boundary<PartitionPosition> left;
>             left = requested.leftBoundary();
>             left = maxLeft(left, sstable.first, true); // second condition
>             if (!isEmpty(left, right))
>                 boundsList.add(AbstractBounds.bounds(left, right));
>         }
>     }
>     else
>     {
>         assert requested.left.compareTo(requested.right) <= 0 || requested.right.isMinimum();
>         Boundary<PartitionPosition> left, right;
>         left = requested.leftBoundary();
>         right = requested.rightBoundary();
>         left = maxLeft(left, sstable.first, true);
>         // apparently isWrapAround() doesn't count Bounds that extend to the limit (min) as wrapping
>         right = requested.right.isMinimum() ? new Boundary<PartitionPosition>(sstable.last, true)
>                                                 : minRight(right, sstable.last, true);
>         if (!isEmpty(left, right))
>             boundsList.add(AbstractBounds.bounds(left, right));
>     }
> }
> {code}
>  * we use token equal ,so isWrapAround is true.
>  * requestd.left = requestd.right = -9223372036854775808,
>  * the real sst dataBoundary.left = -9222849988925915479
>  * so the maxLeft return the real dataBoudary.left. We get the incorrect  data
>  



--
This message was sent by Atlassian Jira
(v8.3.4#803005)

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@cassandra.apache.org
For additional commands, e-mail: commits-help@cassandra.apache.org