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

ASF GitHub Bot commented on FLINK-8679:
---------------------------------------

Github user StefanRRichter commented on a diff in the pull request:

    https://github.com/apache/flink/pull/5518#discussion_r169949298
  
    --- Diff: 
flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/RocksDBKeyedStateBackend.java
 ---
    @@ -1991,43 +2034,87 @@ public int numStateEntries() {
                return count;
        }
     
    -   private static class RocksIteratorWrapper<K> implements Iterator<K> {
    +   /**
    +    * This class is not thread safety.
    +    */
    +   static class RocksIteratorWrapper<K> implements Iterator<K>, 
AutoCloseable {
                private final RocksIterator iterator;
                private final String state;
                private final TypeSerializer<K> keySerializer;
                private final int keyGroupPrefixBytes;
    +           private final byte[] namespaceBytes;
    +           private final boolean ambiguousKeyPossible;
    +           private K nextKey;
     
                public RocksIteratorWrapper(
                                RocksIterator iterator,
                                String state,
                                TypeSerializer<K> keySerializer,
    -                           int keyGroupPrefixBytes) {
    +                           int keyGroupPrefixBytes,
    +                           boolean ambiguousKeyPossible,
    +                           byte[] namespaceBytes) {
                        this.iterator = Preconditions.checkNotNull(iterator);
                        this.state = Preconditions.checkNotNull(state);
                        this.keySerializer = 
Preconditions.checkNotNull(keySerializer);
                        this.keyGroupPrefixBytes = 
Preconditions.checkNotNull(keyGroupPrefixBytes);
    +                   this.namespaceBytes = 
Preconditions.checkNotNull(namespaceBytes);
    +                   this.nextKey = null;
    +                   this.ambiguousKeyPossible = ambiguousKeyPossible;
                }
     
                @Override
                public boolean hasNext() {
    -                   return iterator.isValid();
    +                   final int namespaceBytesLength = namespaceBytes.length;
    +                   final int basicLength = namespaceBytesLength + 
keyGroupPrefixBytes;
    +                   while (nextKey == null && iterator.isValid()) {
    +                           try {
    +                                   byte[] key = iterator.key();
    +                                   if (key.length >= basicLength) {
    --- End diff --
    
    The outer if could become part of the matcher function, or at least we can 
combine bot `if` statements with `&&`.


> RocksDBKeyedBackend.getKeys(stateName, namespace) doesn't filter data with 
> namespace
> ------------------------------------------------------------------------------------
>
>                 Key: FLINK-8679
>                 URL: https://issues.apache.org/jira/browse/FLINK-8679
>             Project: Flink
>          Issue Type: Bug
>          Components: State Backends, Checkpointing
>    Affects Versions: 1.5.0
>            Reporter: Sihua Zhou
>            Assignee: Sihua Zhou
>            Priority: Blocker
>             Fix For: 1.5.0
>
>
> Currently, `RocksDBKeyedBackend.getKeys(stateName, namespace)` is odds. It 
> doesn't use the namespace to filter data. And 
> `HeapKeyedBackend.getKeys(stateName, namespace)` has done that, I think they 
> should be consistent at least.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to