You are viewing a plain text version of this content. The canonical link for it is here.
Posted to github@arrow.apache.org by GitBox <gi...@apache.org> on 2021/05/27 11:49:06 UTC

[GitHub] [arrow] lidavidm commented on a change in pull request #10410: ARROW-10640: [C++] A "where" kernel to combine two arrays based on a mask

lidavidm commented on a change in pull request #10410:
URL: https://github.com/apache/arrow/pull/10410#discussion_r640537055



##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else_test.cc
##########
@@ -0,0 +1,291 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/array.h>
+#include <arrow/compute/api_scalar.h>
+#include <arrow/compute/kernels/test_util.h>
+#include <arrow/testing/gtest_util.h>
+#include <gtest/gtest.h>
+
+namespace arrow {
+namespace compute {
+
+void CheckIfElseOutputArray(const Datum& cond, const Datum& left, const Datum& right,

Review comment:
       I don't think we need `all_valid` given it's pretty much always `false`. It made sense for the original `fill_null` test because you expect no nulls afterwards, but not here. 

##########
File path: cpp/src/arrow/compute/api_scalar.h
##########
@@ -462,5 +462,21 @@ ARROW_EXPORT
 Result<Datum> FillNull(const Datum& values, const Datum& fill_value,
                        ExecContext* ctx = NULLPTR);
 
+/// \brief IfElse returns elements chosen from `left` or `right`
+/// depending on `cond`. `Null` values would be promoted to the result
+///
+/// \param[in] cond `BooleanArray` condition array
+/// \param[in] left scalar/ Array
+/// \param[in] right scalar/ Array
+/// \param[in] ctx the function execution context, optional
+///
+/// \return the resulting datum
+///
+/// \since x.x.x
+/// \note API not yet finalized

Review comment:
       ```suggestion
   /// \brief IfElse returns elements chosen from `left` or `right`
   /// depending on `cond`. `null` values would be promoted to the result
   ///
   /// \param[in] cond `BooleanArray` condition array
   /// \param[in] left scalar/ Array
   /// \param[in] right scalar/ Array
   /// \param[in] ctx the function execution context, optional
   ///
   /// \return the resulting datum
   ///
   /// \since 5.0.0
   /// \note API not yet finalized
   ```

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else.cc
##########
@@ -0,0 +1,838 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/compute/api.h>
+#include <arrow/util/bit_block_counter.h>
+#include <arrow/util/bitmap.h>
+#include <arrow/util/bitmap_ops.h>
+
+#include "codegen_internal.h"
+
+namespace arrow {
+using internal::BitBlockCount;
+using internal::BitBlockCounter;
+using internal::Bitmap;
+
+namespace compute {
+
+namespace {
+
+enum { COND_ALL_VALID = 1, LEFT_ALL_VALID = 2, RIGHT_ALL_VALID = 4 };
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const Scalar& right, ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  if (flag < 6 && flag != 3) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+  }
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // = 7

Review comment:
       Also because RLC/LRC/etc. is a confusing abbreviation especially when the order of arguments is CLR.

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else_test.cc
##########
@@ -0,0 +1,291 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/array.h>
+#include <arrow/compute/api_scalar.h>
+#include <arrow/compute/kernels/test_util.h>
+#include <arrow/testing/gtest_util.h>
+#include <gtest/gtest.h>
+
+namespace arrow {
+namespace compute {
+
+void CheckIfElseOutputArray(const Datum& cond, const Datum& left, const Datum& right,
+                            const Datum& expected, bool all_valid = true) {
+  ASSERT_OK_AND_ASSIGN(Datum datum_out, IfElse(cond, left, right));
+  std::shared_ptr<Array> result = datum_out.make_array();
+  ASSERT_OK(result->ValidateFull());
+  AssertArraysEqual(*expected.make_array(), *result, /*verbose=*/true);
+  if (all_valid) {
+    // Check null count of ArrayData is set, not the computed Array.null_count
+    ASSERT_EQ(result->data()->null_count, 0);
+  }
+}
+
+void CheckIfElseOutputAAA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputAAS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right, expected_, all_valid);
+}
+
+void CheckIfElseOutputASA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputASS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left,
+                          const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right, expected_, all_valid);
+}
+
+class TestIfElseKernel : public ::testing::Test {};
+
+template <typename Type>
+class TestIfElsePrimitive : public ::testing::Test {};
+
+using PrimitiveTypes = ::testing::Types<Int8Type, UInt8Type, Int16Type, UInt16Type,
+                                        Int32Type, UInt32Type, Int64Type, UInt64Type,
+                                        FloatType, DoubleType, Date32Type, Date64Type>;
+
+TYPED_TEST_SUITE(TestIfElsePrimitive, PrimitiveTypes);
+
+TYPED_TEST(TestIfElsePrimitive, IfElseFixedSize) {
+  auto type = TypeTraits<TypeParam>::type_singleton();
+
+  // No Nulls
+  CheckIfElseOutputAAA(type, "[]", "[]", "[]", "[]");
+
+  // -------- All arrays ---------
+  // RLC = 111
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, 3, 8]");
+  // RLC = 110
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, null, 8]", false);
+  // RLC = 101
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, 3, 8]", false);
+  // RLC = 100
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, null, 8]", false);
+  // RLC = 011
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[1, 2, 3, null]", false);
+  // RLC = 010
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[null, 2, 3, null]", false);
+  // RLC = 001
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[1, 2, null, null]", false);
+  // RLC = 000
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[null, 2, null, null]", false);
+
+  using ArrayType = typename TypeTraits<TypeParam>::ArrayType;
+  random::RandomArrayGenerator rand(/*seed=*/0);
+  int64_t len = 1000;
+  auto cond = std::static_pointer_cast<BooleanArray>(
+      rand.ArrayOf(boolean(), len, /*null_probability=*/0.01));
+  auto left = std::static_pointer_cast<ArrayType>(
+      rand.ArrayOf(type, len, /*null_probability=*/0.01));
+  auto right = std::static_pointer_cast<ArrayType>(
+      rand.ArrayOf(type, len, /*null_probability=*/0.01));
+
+  typename TypeTraits<TypeParam>::BuilderType builder;
+
+  for (int64_t i = 0; i < len; ++i) {
+    if (!cond->IsValid(i) || (cond->Value(i) && !left->IsValid(i)) ||
+        (!cond->Value(i) && !right->IsValid(i))) {
+      ASSERT_OK(builder.AppendNull());
+      continue;
+    }
+
+    if (cond->Value(i)) {
+      ASSERT_OK(builder.Append(left->Value(i)));
+    } else {
+      ASSERT_OK(builder.Append(right->Value(i)));
+    }
+  }
+  ASSERT_OK_AND_ASSIGN(auto expected_data, builder.Finish());
+
+  CheckIfElseOutputArray(cond, left, right, expected_data, false);
+
+  // -------- Cond - Array, Left- Array, Right - Scalar ---------
+
+  ASSERT_OK_AND_ASSIGN(std::shared_ptr<Scalar> valid_scalar, MakeScalar(type, 100));
+  std::shared_ptr<Scalar> null_scalar = MakeNullScalar(type);
+
+  // empty
+  CheckIfElseOutputAAS(type, "[]", "[]", valid_scalar, "[]");
+
+  // RLC = 111
+  CheckIfElseOutputAAS(type, "[true, true, true, false]", "[1, 2, 3, 4]", valid_scalar,
+                       "[1, 2, 3, 100]");
+  // RLC = 110
+  CheckIfElseOutputAAS(type, "[true, true, null, false]", "[1, 2, 3, 4]", valid_scalar,
+                       "[1, 2, null, 100]", false);
+  // RLC = 101
+  CheckIfElseOutputAAS(type, "[true, true, true, false]", "[1, null, 3, 4]", valid_scalar,
+                       "[1, null, 3, 100]", false);
+  // RLC = 100
+  CheckIfElseOutputAAS(type, "[true, true, null, false]", "[1, null, 3, 4]", valid_scalar,
+                       "[1, null, null, 100]", false);

Review comment:
       It's also unclear to me what RLC/LRC stand for.

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else.cc
##########
@@ -0,0 +1,838 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/compute/api.h>
+#include <arrow/util/bit_block_counter.h>
+#include <arrow/util/bitmap.h>
+#include <arrow/util/bitmap_ops.h>
+
+#include "codegen_internal.h"
+
+namespace arrow {
+using internal::BitBlockCount;
+using internal::BitBlockCounter;
+using internal::Bitmap;
+
+namespace compute {
+
+namespace {
+
+enum { COND_ALL_VALID = 1, LEFT_ALL_VALID = 2, RIGHT_ALL_VALID = 4 };
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const Scalar& right, ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  if (flag < 6 && flag != 3) {

Review comment:
       This is a little confusing: can we move this into the switch below as another case?

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else.cc
##########
@@ -0,0 +1,838 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/compute/api.h>
+#include <arrow/util/bit_block_counter.h>
+#include <arrow/util/bitmap.h>
+#include <arrow/util/bitmap_ops.h>
+
+#include "codegen_internal.h"
+
+namespace arrow {
+using internal::BitBlockCount;
+using internal::BitBlockCounter;
+using internal::Bitmap;
+
+namespace compute {
+
+namespace {
+
+enum { COND_ALL_VALID = 1, LEFT_ALL_VALID = 2, RIGHT_ALL_VALID = 4 };
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const Scalar& right, ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  if (flag < 6 && flag != 3) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+  }
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // = 7

Review comment:
       nit: below you abbreviate these as RLC/LRC/etc. I don't think we need any of those comments since the case itself should be self explanatory.

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else.cc
##########
@@ -0,0 +1,838 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/compute/api.h>
+#include <arrow/util/bit_block_counter.h>
+#include <arrow/util/bitmap.h>
+#include <arrow/util/bitmap_ops.h>
+
+#include "codegen_internal.h"
+
+namespace arrow {
+using internal::BitBlockCount;
+using internal::BitBlockCounter;
+using internal::Bitmap;
+
+namespace compute {
+
+namespace {
+
+enum { COND_ALL_VALID = 1, LEFT_ALL_VALID = 2, RIGHT_ALL_VALID = 4 };
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const Scalar& right, ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  if (flag < 6 && flag != 3) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+  }
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // = 7
+      break;
+    case LEFT_ALL_VALID | RIGHT_ALL_VALID:  // = 6
+      // out_valid = c_valid
+      output->buffers[0] = SliceBuffer(cond.buffers[0], cond.offset, cond.length);
+      break;
+    case COND_ALL_VALID | RIGHT_ALL_VALID:  // = 5
+      // out_valid = ~cond.data
+      arrow::internal::InvertBitmap(cond.buffers[1]->data(), cond.offset, cond.length,
+                                    output->buffers[0]->mutable_data(), 0);
+      break;
+    case RIGHT_ALL_VALID:  // = 4
+      // out_valid = c_valid & ~cond.data
+      arrow::internal::BitmapAndNot(cond.buffers[0]->data(), cond.offset,
+                                    cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                    output->buffers[0]->mutable_data());
+      break;
+    case COND_ALL_VALID | LEFT_ALL_VALID:  // = 3
+      // out_valid = cond.data
+      output->buffers[0] = SliceBuffer(cond.buffers[1], cond.offset, cond.length);
+      break;
+    case LEFT_ALL_VALID:  // = 2
+      // out_valid = cond.valid & cond.data
+      arrow::internal::BitmapAnd(cond.buffers[0]->data(), cond.offset,
+                                 cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                 output->buffers[0]->mutable_data());
+      break;
+    case COND_ALL_VALID:  // = 1
+      // out_valid = 0 --> nothing to do; but requires out_valid to be a all-zero buffer

Review comment:
       AllocateBitmap doesn't zero the allocation
   https://github.com/apache/arrow/blob/176988893e182ac418072ef8cd9a4bc598784d97/cpp/src/arrow/compute/kernel.h#L61-L64

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else_test.cc
##########
@@ -0,0 +1,291 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/array.h>
+#include <arrow/compute/api_scalar.h>
+#include <arrow/compute/kernels/test_util.h>
+#include <arrow/testing/gtest_util.h>
+#include <gtest/gtest.h>
+
+namespace arrow {
+namespace compute {
+
+void CheckIfElseOutputArray(const Datum& cond, const Datum& left, const Datum& right,
+                            const Datum& expected, bool all_valid = true) {
+  ASSERT_OK_AND_ASSIGN(Datum datum_out, IfElse(cond, left, right));
+  std::shared_ptr<Array> result = datum_out.make_array();
+  ASSERT_OK(result->ValidateFull());
+  AssertArraysEqual(*expected.make_array(), *result, /*verbose=*/true);
+  if (all_valid) {
+    // Check null count of ArrayData is set, not the computed Array.null_count
+    ASSERT_EQ(result->data()->null_count, 0);
+  }
+}
+
+void CheckIfElseOutputAAA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputAAS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right, expected_, all_valid);
+}
+
+void CheckIfElseOutputASA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputASS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left,
+                          const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right, expected_, all_valid);
+}
+
+class TestIfElseKernel : public ::testing::Test {};
+
+template <typename Type>
+class TestIfElsePrimitive : public ::testing::Test {};
+
+using PrimitiveTypes = ::testing::Types<Int8Type, UInt8Type, Int16Type, UInt16Type,
+                                        Int32Type, UInt32Type, Int64Type, UInt64Type,
+                                        FloatType, DoubleType, Date32Type, Date64Type>;
+
+TYPED_TEST_SUITE(TestIfElsePrimitive, PrimitiveTypes);
+
+TYPED_TEST(TestIfElsePrimitive, IfElseFixedSize) {
+  auto type = TypeTraits<TypeParam>::type_singleton();
+
+  // No Nulls
+  CheckIfElseOutputAAA(type, "[]", "[]", "[]", "[]");
+
+  // -------- All arrays ---------
+  // RLC = 111
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, 3, 8]");
+  // RLC = 110
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, null, 8]", false);
+  // RLC = 101
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, 3, 8]", false);
+  // RLC = 100
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, null, 8]", false);
+  // RLC = 011
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[1, 2, 3, null]", false);
+  // RLC = 010
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[null, 2, 3, null]", false);
+  // RLC = 001
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[1, 2, null, null]", false);
+  // RLC = 000
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[null, 2, null, null]", false);
+
+  using ArrayType = typename TypeTraits<TypeParam>::ArrayType;
+  random::RandomArrayGenerator rand(/*seed=*/0);
+  int64_t len = 1000;
+  auto cond = std::static_pointer_cast<BooleanArray>(
+      rand.ArrayOf(boolean(), len, /*null_probability=*/0.01));
+  auto left = std::static_pointer_cast<ArrayType>(
+      rand.ArrayOf(type, len, /*null_probability=*/0.01));
+  auto right = std::static_pointer_cast<ArrayType>(
+      rand.ArrayOf(type, len, /*null_probability=*/0.01));
+
+  typename TypeTraits<TypeParam>::BuilderType builder;
+
+  for (int64_t i = 0; i < len; ++i) {
+    if (!cond->IsValid(i) || (cond->Value(i) && !left->IsValid(i)) ||
+        (!cond->Value(i) && !right->IsValid(i))) {
+      ASSERT_OK(builder.AppendNull());
+      continue;
+    }
+
+    if (cond->Value(i)) {
+      ASSERT_OK(builder.Append(left->Value(i)));
+    } else {
+      ASSERT_OK(builder.Append(right->Value(i)));
+    }
+  }
+  ASSERT_OK_AND_ASSIGN(auto expected_data, builder.Finish());
+
+  CheckIfElseOutputArray(cond, left, right, expected_data, false);
+
+  // -------- Cond - Array, Left- Array, Right - Scalar ---------
+
+  ASSERT_OK_AND_ASSIGN(std::shared_ptr<Scalar> valid_scalar, MakeScalar(type, 100));
+  std::shared_ptr<Scalar> null_scalar = MakeNullScalar(type);
+
+  // empty
+  CheckIfElseOutputAAS(type, "[]", "[]", valid_scalar, "[]");
+
+  // RLC = 111
+  CheckIfElseOutputAAS(type, "[true, true, true, false]", "[1, 2, 3, 4]", valid_scalar,
+                       "[1, 2, 3, 100]");
+  // RLC = 110
+  CheckIfElseOutputAAS(type, "[true, true, null, false]", "[1, 2, 3, 4]", valid_scalar,
+                       "[1, 2, null, 100]", false);
+  // RLC = 101
+  CheckIfElseOutputAAS(type, "[true, true, true, false]", "[1, null, 3, 4]", valid_scalar,
+                       "[1, null, 3, 100]", false);
+  // RLC = 100
+  CheckIfElseOutputAAS(type, "[true, true, null, false]", "[1, null, 3, 4]", valid_scalar,
+                       "[1, null, null, 100]", false);

Review comment:
       Couldn't these cases be combined? e.g. `[true, true, null, null, false, false]`, `[1, null, 2, null, 3, null]`, and `valid_scalar` -> `[1, null, null, null, 100, null]`?
   
   And ditto overall.

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else_test.cc
##########
@@ -0,0 +1,291 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/array.h>
+#include <arrow/compute/api_scalar.h>
+#include <arrow/compute/kernels/test_util.h>
+#include <arrow/testing/gtest_util.h>
+#include <gtest/gtest.h>
+
+namespace arrow {
+namespace compute {
+
+void CheckIfElseOutputArray(const Datum& cond, const Datum& left, const Datum& right,
+                            const Datum& expected, bool all_valid = true) {
+  ASSERT_OK_AND_ASSIGN(Datum datum_out, IfElse(cond, left, right));
+  std::shared_ptr<Array> result = datum_out.make_array();
+  ASSERT_OK(result->ValidateFull());
+  AssertArraysEqual(*expected.make_array(), *result, /*verbose=*/true);
+  if (all_valid) {
+    // Check null count of ArrayData is set, not the computed Array.null_count
+    ASSERT_EQ(result->data()->null_count, 0);
+  }
+}
+
+void CheckIfElseOutputAAA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputAAS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right, expected_, all_valid);
+}
+
+void CheckIfElseOutputASA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputASS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left,
+                          const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right, expected_, all_valid);
+}
+
+class TestIfElseKernel : public ::testing::Test {};
+
+template <typename Type>
+class TestIfElsePrimitive : public ::testing::Test {};
+
+using PrimitiveTypes = ::testing::Types<Int8Type, UInt8Type, Int16Type, UInt16Type,
+                                        Int32Type, UInt32Type, Int64Type, UInt64Type,
+                                        FloatType, DoubleType, Date32Type, Date64Type>;
+
+TYPED_TEST_SUITE(TestIfElsePrimitive, PrimitiveTypes);
+
+TYPED_TEST(TestIfElsePrimitive, IfElseFixedSize) {
+  auto type = TypeTraits<TypeParam>::type_singleton();
+
+  // No Nulls
+  CheckIfElseOutputAAA(type, "[]", "[]", "[]", "[]");
+
+  // -------- All arrays ---------
+  // RLC = 111
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, 3, 8]");
+  // RLC = 110
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, null, 8]", false);
+  // RLC = 101
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, 3, 8]", false);
+  // RLC = 100
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, null, 8]", false);
+  // RLC = 011
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[1, 2, 3, null]", false);
+  // RLC = 010
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[null, 2, 3, null]", false);
+  // RLC = 001
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[1, 2, null, null]", false);
+  // RLC = 000
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[null, 2, null, null]", false);
+
+  using ArrayType = typename TypeTraits<TypeParam>::ArrayType;

Review comment:
       nit: do you mind extracting out the random array test into a separate test case?

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else.cc
##########
@@ -0,0 +1,838 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/compute/api.h>
+#include <arrow/util/bit_block_counter.h>
+#include <arrow/util/bitmap.h>
+#include <arrow/util/bitmap_ops.h>
+
+#include "codegen_internal.h"
+
+namespace arrow {
+using internal::BitBlockCount;
+using internal::BitBlockCounter;
+using internal::Bitmap;
+
+namespace compute {
+
+namespace {
+
+enum { COND_ALL_VALID = 1, LEFT_ALL_VALID = 2, RIGHT_ALL_VALID = 4 };
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const Scalar& right, ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  if (flag < 6 && flag != 3) {

Review comment:
       You could move the allocation into the cases as well.

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else_test.cc
##########
@@ -0,0 +1,291 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/array.h>
+#include <arrow/compute/api_scalar.h>
+#include <arrow/compute/kernels/test_util.h>
+#include <arrow/testing/gtest_util.h>
+#include <gtest/gtest.h>
+
+namespace arrow {
+namespace compute {
+
+void CheckIfElseOutputArray(const Datum& cond, const Datum& left, const Datum& right,
+                            const Datum& expected, bool all_valid = true) {
+  ASSERT_OK_AND_ASSIGN(Datum datum_out, IfElse(cond, left, right));
+  std::shared_ptr<Array> result = datum_out.make_array();
+  ASSERT_OK(result->ValidateFull());
+  AssertArraysEqual(*expected.make_array(), *result, /*verbose=*/true);
+  if (all_valid) {
+    // Check null count of ArrayData is set, not the computed Array.null_count
+    ASSERT_EQ(result->data()->null_count, 0);
+  }
+}
+
+void CheckIfElseOutputAAA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputAAS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::string& left, const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& left_ = ArrayFromJSON(type, left);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left_, right, expected_, all_valid);
+}
+
+void CheckIfElseOutputASA(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left, const std::string& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& right_ = ArrayFromJSON(type, right);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right_, expected_, all_valid);
+}
+
+void CheckIfElseOutputASS(const std::shared_ptr<DataType>& type, const std::string& cond,
+                          const std::shared_ptr<Scalar>& left,
+                          const std::shared_ptr<Scalar>& right,
+                          const std::string& expected, bool all_valid = true) {
+  const std::shared_ptr<Array>& cond_ = ArrayFromJSON(boolean(), cond);
+  const std::shared_ptr<Array>& expected_ = ArrayFromJSON(type, expected);
+  CheckIfElseOutputArray(cond_, left, right, expected_, all_valid);
+}
+
+class TestIfElseKernel : public ::testing::Test {};
+
+template <typename Type>
+class TestIfElsePrimitive : public ::testing::Test {};
+
+using PrimitiveTypes = ::testing::Types<Int8Type, UInt8Type, Int16Type, UInt16Type,
+                                        Int32Type, UInt32Type, Int64Type, UInt64Type,
+                                        FloatType, DoubleType, Date32Type, Date64Type>;
+
+TYPED_TEST_SUITE(TestIfElsePrimitive, PrimitiveTypes);
+
+TYPED_TEST(TestIfElsePrimitive, IfElseFixedSize) {
+  auto type = TypeTraits<TypeParam>::type_singleton();
+
+  // No Nulls
+  CheckIfElseOutputAAA(type, "[]", "[]", "[]", "[]");
+
+  // -------- All arrays ---------
+  // RLC = 111
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, 3, 8]");
+  // RLC = 110
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, 2, 3, 4]", "[5, 6, 7, 8]",
+                       "[1, 2, null, 8]", false);
+  // RLC = 101
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, 3, 8]", false);
+  // RLC = 100
+  CheckIfElseOutputAAA(type, "[true, true, null, false]", "[1, null, 3, 4]",
+                       "[5, 6, 7, 8]", "[1, null, null, 8]", false);
+  // RLC = 011
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[1, 2, 3, null]", false);
+  // RLC = 010
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, 3, 4]",
+                       "[5, 6, 7, null]", "[null, 2, 3, null]", false);
+  // RLC = 001
+  CheckIfElseOutputAAA(type, "[true, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[1, 2, null, null]", false);
+  // RLC = 000
+  CheckIfElseOutputAAA(type, "[null, true, true, false]", "[1, 2, null, null]",
+                       "[null, 6, 7, null]", "[null, 2, null, null]", false);
+
+  using ArrayType = typename TypeTraits<TypeParam>::ArrayType;
+  random::RandomArrayGenerator rand(/*seed=*/0);
+  int64_t len = 1000;
+  auto cond = std::static_pointer_cast<BooleanArray>(
+      rand.ArrayOf(boolean(), len, /*null_probability=*/0.01));
+  auto left = std::static_pointer_cast<ArrayType>(
+      rand.ArrayOf(type, len, /*null_probability=*/0.01));
+  auto right = std::static_pointer_cast<ArrayType>(
+      rand.ArrayOf(type, len, /*null_probability=*/0.01));
+
+  typename TypeTraits<TypeParam>::BuilderType builder;
+
+  for (int64_t i = 0; i < len; ++i) {
+    if (!cond->IsValid(i) || (cond->Value(i) && !left->IsValid(i)) ||
+        (!cond->Value(i) && !right->IsValid(i))) {
+      ASSERT_OK(builder.AppendNull());
+      continue;
+    }
+
+    if (cond->Value(i)) {
+      ASSERT_OK(builder.Append(left->Value(i)));
+    } else {
+      ASSERT_OK(builder.Append(right->Value(i)));
+    }
+  }
+  ASSERT_OK_AND_ASSIGN(auto expected_data, builder.Finish());
+
+  CheckIfElseOutputArray(cond, left, right, expected_data, false);
+
+  // -------- Cond - Array, Left- Array, Right - Scalar ---------
+
+  ASSERT_OK_AND_ASSIGN(std::shared_ptr<Scalar> valid_scalar, MakeScalar(type, 100));
+  std::shared_ptr<Scalar> null_scalar = MakeNullScalar(type);
+
+  // empty
+  CheckIfElseOutputAAS(type, "[]", "[]", valid_scalar, "[]");
+
+  // RLC = 111
+  CheckIfElseOutputAAS(type, "[true, true, true, false]", "[1, 2, 3, 4]", valid_scalar,
+                       "[1, 2, 3, 100]");
+  // RLC = 110
+  CheckIfElseOutputAAS(type, "[true, true, null, false]", "[1, 2, 3, 4]", valid_scalar,
+                       "[1, 2, null, 100]", false);
+  // RLC = 101
+  CheckIfElseOutputAAS(type, "[true, true, true, false]", "[1, null, 3, 4]", valid_scalar,
+                       "[1, null, 3, 100]", false);
+  // RLC = 100
+  CheckIfElseOutputAAS(type, "[true, true, null, false]", "[1, null, 3, 4]", valid_scalar,
+                       "[1, null, null, 100]", false);

Review comment:
       Ok, I see from above that it's about the validity of cond/left/right. IMO if you're going to abbreviate it, at least always write it as C/L/R since that's the order of the arguments.

##########
File path: cpp/src/arrow/compute/kernels/scalar_if_else.cc
##########
@@ -0,0 +1,838 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+#include <arrow/compute/api.h>
+#include <arrow/util/bit_block_counter.h>
+#include <arrow/util/bitmap.h>
+#include <arrow/util/bitmap_ops.h>
+
+#include "codegen_internal.h"
+
+namespace arrow {
+using internal::BitBlockCount;
+using internal::BitBlockCounter;
+using internal::Bitmap;
+
+namespace compute {
+
+namespace {
+
+enum { COND_ALL_VALID = 1, LEFT_ALL_VALID = 2, RIGHT_ALL_VALID = 4 };
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const Scalar& right, ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  if (flag < 6 && flag != 3) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+  }
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // = 7
+      break;
+    case LEFT_ALL_VALID | RIGHT_ALL_VALID:  // = 6
+      // out_valid = c_valid
+      output->buffers[0] = SliceBuffer(cond.buffers[0], cond.offset, cond.length);
+      break;
+    case COND_ALL_VALID | RIGHT_ALL_VALID:  // = 5
+      // out_valid = ~cond.data
+      arrow::internal::InvertBitmap(cond.buffers[1]->data(), cond.offset, cond.length,
+                                    output->buffers[0]->mutable_data(), 0);
+      break;
+    case RIGHT_ALL_VALID:  // = 4
+      // out_valid = c_valid & ~cond.data
+      arrow::internal::BitmapAndNot(cond.buffers[0]->data(), cond.offset,
+                                    cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                    output->buffers[0]->mutable_data());
+      break;
+    case COND_ALL_VALID | LEFT_ALL_VALID:  // = 3
+      // out_valid = cond.data
+      output->buffers[0] = SliceBuffer(cond.buffers[1], cond.offset, cond.length);
+      break;
+    case LEFT_ALL_VALID:  // = 2
+      // out_valid = cond.valid & cond.data
+      arrow::internal::BitmapAnd(cond.buffers[0]->data(), cond.offset,
+                                 cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                 output->buffers[0]->mutable_data());
+      break;
+    case COND_ALL_VALID:  // = 1
+      // out_valid = 0 --> nothing to do; but requires out_valid to be a all-zero buffer
+      break;
+    case 0:  // RLC = 000
+      // out_valid = 0 --> nothing to do; but requires out_valid to be a all-zero buffer
+      break;
+  }
+  return Status::OK();
+}
+
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond,
+                           const ArrayData& left, const Scalar& right,
+                           ArrayData* output) {
+  uint8_t flag = right.is_valid * 4 + !left.MayHaveNulls() * 2 + !cond.MayHaveNulls();
+
+  enum { C_VALID, C_DATA, L_VALID };
+
+  Bitmap bitmaps[3];
+  bitmaps[C_VALID] = {cond.buffers[0], cond.offset, cond.length};
+  bitmaps[C_DATA] = {cond.buffers[1], cond.offset, cond.length};
+  bitmaps[L_VALID] = {left.buffers[0], left.offset, left.length};
+
+  uint64_t* out_validity = nullptr;
+  if (flag < 6 && flag != 3) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+    out_validity = output->GetMutableValues<uint64_t>(0);
+  }
+
+  // lambda function that will be used inside the visitor
+  int64_t i = 0;
+  auto apply = [&](uint64_t c_valid, uint64_t c_data, uint64_t l_valid,
+                   uint64_t r_valid) {
+    out_validity[i] = c_valid & ((c_data & l_valid) | (~c_data & r_valid));
+    i++;
+  };
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 111
+      break;
+    case LEFT_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 110
+      output->buffers[0] = SliceBuffer(cond.buffers[0], cond.offset, cond.length);
+      break;
+    case COND_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 101
+      // bitmaps[C_VALID] might be null; override to make it safe for Visit()
+      bitmaps[C_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 3> words) {
+        apply(UINT64_MAX, words[C_DATA], words[L_VALID], UINT64_MAX);
+      });
+      break;
+    case RIGHT_ALL_VALID:  // RLC = 100
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 3> words) {
+        apply(words[C_VALID], words[C_DATA], words[L_VALID], UINT64_MAX);
+      });
+      break;
+    case COND_ALL_VALID | LEFT_ALL_VALID:  // RLC = 011
+      // only cond.data is passed
+      output->buffers[0] = SliceBuffer(cond.buffers[1], cond.offset, cond.length);
+      break;
+    case LEFT_ALL_VALID:  // RLC = 010
+      // out_valid = cond.valid & cond.data
+      arrow::internal::BitmapAnd(cond.buffers[0]->data(), cond.offset,
+                                 cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                 output->buffers[0]->mutable_data());
+      break;
+    case COND_ALL_VALID:  // RLC = 001
+      // out_valid = cond.data & left.valid
+      arrow::internal::BitmapAnd(cond.buffers[1]->data(), cond.offset,
+                                 left.buffers[0]->data(), left.offset, cond.length, 0,
+                                 output->buffers[0]->mutable_data());
+      break;
+    case 0:  // RLC = 000
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 3> words) {
+        apply(words[C_VALID], words[C_DATA], words[L_VALID], 0);
+      });
+      break;
+  }
+  return Status::OK();
+}
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                           const ArrayData& right, ArrayData* output) {
+  uint8_t flag = !right.MayHaveNulls() * 4 + left.is_valid * 2 + !cond.MayHaveNulls();
+
+  enum { C_VALID, C_DATA, R_VALID };
+
+  Bitmap bitmaps[3];
+  bitmaps[C_VALID] = {cond.buffers[0], cond.offset, cond.length};
+  bitmaps[C_DATA] = {cond.buffers[1], cond.offset, cond.length};
+  bitmaps[R_VALID] = {right.buffers[0], right.offset, right.length};
+
+  uint64_t* out_validity = nullptr;
+  if (flag < 6) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+    out_validity = output->GetMutableValues<uint64_t>(0);
+  }
+
+  // lambda function that will be used inside the visitor
+  int64_t i = 0;
+  auto apply = [&](uint64_t c_valid, uint64_t c_data, uint64_t l_valid,
+                   uint64_t r_valid) {
+    out_validity[i] = c_valid & ((c_data & l_valid) | (~c_data & r_valid));
+    i++;
+  };
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 111
+      break;
+    case LEFT_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 110
+      output->buffers[0] = SliceBuffer(cond.buffers[0], cond.offset, cond.length);
+      break;
+    case COND_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 101
+      // out_valid = ~cond.data
+      arrow::internal::InvertBitmap(cond.buffers[1]->data(), cond.offset, cond.length,
+                                    output->buffers[0]->mutable_data(), 0);
+      break;
+    case RIGHT_ALL_VALID:  // RLC = 100
+      // out_valid = c_valid & ~cond.data
+      arrow::internal::BitmapAndNot(cond.buffers[0]->data(), cond.offset,
+                                    cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                    output->buffers[0]->mutable_data());
+      break;
+    case COND_ALL_VALID | LEFT_ALL_VALID:  // RLC = 011
+      // bitmaps[C_VALID] might be null; override to make it safe for Visit()
+      bitmaps[C_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 3> words) {
+        apply(UINT64_MAX, words[C_DATA], UINT64_MAX, words[R_VALID]);
+      });
+      break;
+    case LEFT_ALL_VALID:  // RLC = 010
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 3> words) {
+        apply(words[C_VALID], words[C_DATA], UINT64_MAX, words[R_VALID]);
+      });
+      break;
+    case COND_ALL_VALID:  // RLC = 001
+      // out_valid =  ~cond.data & right.valid
+      arrow::internal::BitmapAndNot(right.buffers[0]->data(), right.offset,
+                                    cond.buffers[1]->data(), cond.offset, cond.length, 0,
+                                    output->buffers[0]->mutable_data());
+      break;
+    case 0:  // RLC = 000
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 3> words) {
+        apply(words[C_VALID], words[C_DATA], 0, words[R_VALID]);
+      });
+      break;
+  }
+  return Status::OK();
+}
+
+// if the condition is null then output is null otherwise we take validity from the
+// selected argument
+// ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+Status PromoteNullsVisitor(KernelContext* ctx, const ArrayData& cond,
+                           const ArrayData& left, const ArrayData& right,
+                           ArrayData* output) {
+  uint8_t flag =
+      !right.MayHaveNulls() * 4 + !left.MayHaveNulls() * 2 + !cond.MayHaveNulls();
+
+  enum { C_VALID, C_DATA, L_VALID, R_VALID };
+
+  Bitmap bitmaps[4];
+  bitmaps[C_VALID] = {cond.buffers[0], cond.offset, cond.length};
+  bitmaps[C_DATA] = {cond.buffers[1], cond.offset, cond.length};
+  bitmaps[L_VALID] = {left.buffers[0], left.offset, left.length};
+  bitmaps[R_VALID] = {right.buffers[0], right.offset, right.length};
+
+  uint64_t* out_validity = nullptr;
+  if (flag < 6) {
+    // there will be a validity buffer in the output
+    ARROW_ASSIGN_OR_RAISE(output->buffers[0], ctx->AllocateBitmap(cond.length));
+    out_validity = output->GetMutableValues<uint64_t>(0);
+  }
+
+  // lambda function that will be used inside the visitor
+  int64_t i = 0;
+  auto apply = [&](uint64_t c_valid, uint64_t c_data, uint64_t l_valid,
+                   uint64_t r_valid) {
+    out_validity[i] = c_valid & ((c_data & l_valid) | (~c_data & r_valid));
+    i++;
+  };
+
+  // if the condition is null then output is null otherwise we take validity from the
+  // selected argument
+  // ie. cond.valid & (cond.data & left.valid | ~cond.data & right.valid)
+  switch (flag) {
+    case COND_ALL_VALID | LEFT_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 111
+      break;
+    case LEFT_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 110
+      output->buffers[0] = SliceBuffer(cond.buffers[0], cond.offset, cond.length);
+      break;
+    case COND_ALL_VALID | RIGHT_ALL_VALID:  // RLC = 101
+      // bitmaps[C_VALID], bitmaps[R_VALID] might be null; override to make it safe for
+      // Visit()
+      bitmaps[C_VALID] = bitmaps[C_DATA];
+      bitmaps[R_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 4> words) {
+        apply(UINT64_MAX, words[C_DATA], words[L_VALID], UINT64_MAX);
+      });
+      break;
+    case RIGHT_ALL_VALID:  // RLC = 100
+      // bitmaps[R_VALID] might be null; override to make it safe for Visit()
+      bitmaps[R_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 4> words) {
+        apply(words[C_VALID], words[C_DATA], words[L_VALID], UINT64_MAX);
+      });
+      break;
+    case COND_ALL_VALID | LEFT_ALL_VALID:  // RLC = 011
+      // bitmaps[C_VALID], bitmaps[L_VALID] might be null; override to make it safe for
+      // Visit()
+      bitmaps[C_VALID] = bitmaps[C_DATA];
+      bitmaps[L_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 4> words) {
+        apply(UINT64_MAX, words[C_DATA], UINT64_MAX, words[R_VALID]);
+      });
+      break;
+    case LEFT_ALL_VALID:  // RLC = 010
+      // bitmaps[L_VALID] might be null; override to make it safe for Visit()
+      bitmaps[L_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 4> words) {
+        apply(words[C_VALID], words[C_DATA], UINT64_MAX, words[R_VALID]);
+      });
+      break;
+    case COND_ALL_VALID:  // RLC = 001
+      // bitmaps[C_VALID] might be null; override to make it safe for Visit()
+      bitmaps[C_VALID] = bitmaps[C_DATA];
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 4> words) {
+        apply(UINT64_MAX, words[C_DATA], words[L_VALID], words[R_VALID]);
+      });
+      break;
+    case 0:  // RLC = 000
+      Bitmap::VisitWords(bitmaps, [&](std::array<uint64_t, 4> words) {
+        apply(words[C_VALID], words[C_DATA], words[L_VALID], words[R_VALID]);
+      });
+      break;
+  }
+  return Status::OK();
+}
+
+// nulls will be promoted as follows:
+// cond.valid && (cond.data && left.valid || ~cond.data && right.valid)
+// Note: we have to work on ArrayData. Otherwise we won't be able to handle array
+// offsets AAA
+/*Status PromoteNulls(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                    const ArrayData& right, ArrayData* output) {
+  if (!cond.MayHaveNulls() && !left.MayHaveNulls() && !right.MayHaveNulls()) {
+    return Status::OK();  // no nulls to handle
+  }
+  const int64_t len = cond.length;
+
+  // out_validity = ~cond.data --> mask right values
+  ARROW_ASSIGN_OR_RAISE(
+      std::shared_ptr<Buffer> out_validity,
+      arrow::internal::InvertBitmap(ctx->memory_pool(), cond.buffers[1]->data(),
+                                    cond.offset, len));
+
+  if (right.MayHaveNulls()) {  // out_validity = right.valid && ~cond.data
+    arrow::internal::BitmapAnd(right.buffers[0]->data(), right.offset,
+                               out_validity->data(), 0, len, 0,
+                               out_validity->mutable_data());
+  }
+
+  std::shared_ptr<Buffer> tmp_buf;
+  if (left.MayHaveNulls()) {
+    // tmp_buf = left.valid && cond.data
+    ARROW_ASSIGN_OR_RAISE(
+        tmp_buf, arrow::internal::BitmapAnd(ctx->memory_pool(), left.buffers[0]->data(),
+                                            left.offset, cond.buffers[1]->data(),
+                                            cond.offset, len, 0));
+  } else {  // if left all valid --> tmp_buf = cond.data (zero copy slice)
+    tmp_buf = SliceBuffer(cond.buffers[1], cond.offset, cond.length);
+  }
+
+  // out_validity = cond.data && left.valid || ~cond.data && right.valid
+  arrow::internal::BitmapOr(out_validity->data(), 0, tmp_buf->data(), 0, len, 0,
+                            out_validity->mutable_data());
+
+  if (cond.MayHaveNulls()) {
+    // out_validity = cond.valid && (cond.data && left.valid || ~cond.data && right.valid)
+    ::arrow::internal::BitmapAnd(out_validity->data(), 0, cond.buffers[0]->data(),
+                                 cond.offset, len, 0, out_validity->mutable_data());
+  }
+
+  output->buffers[0] = std::move(out_validity);
+  output->GetNullCount();  // update null count
+  return Status::OK();
+}
+
+// cond.valid && (cond.data && left.valid || ~cond.data && right.valid)
+// ASA and AAS
+Status PromoteNulls(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                    const ArrayData& right, ArrayData* output) {
+  if (!cond.MayHaveNulls() && left.is_valid && !right.MayHaveNulls()) {
+    return Status::OK();  // no nulls to handle
+  }
+  const int64_t len = cond.length;
+
+  // out_validity = ~cond.data
+  ARROW_ASSIGN_OR_RAISE(
+      std::shared_ptr<Buffer> out_validity,
+      arrow::internal::InvertBitmap(ctx->memory_pool(), cond.buffers[1]->data(),
+                                    cond.offset, len));
+  // out_validity = ~cond.data && right.valid
+  if (right.MayHaveNulls()) {  // out_validity = right.valid && ~cond.data
+    arrow::internal::BitmapAnd(right.buffers[0]->data(), right.offset,
+                               out_validity->data(), 0, len, 0,
+                               out_validity->mutable_data());
+  }
+
+  // out_validity = cond.data && left.valid || ~cond.data && right.valid
+  if (left.is_valid) {
+    arrow::internal::BitmapOr(out_validity->data(), 0, cond.buffers[1]->data(),
+                              cond.offset, len, 0, out_validity->mutable_data());
+  }
+
+  // out_validity = cond.valid && (cond.data && left.valid || ~cond.data && right.valid)
+  if (cond.MayHaveNulls()) {
+    ::arrow::internal::BitmapAnd(out_validity->data(), 0, cond.buffers[0]->data(),
+                                 cond.offset, len, 0, out_validity->mutable_data());
+  }
+
+  output->buffers[0] = std::move(out_validity);
+  output->GetNullCount();  // update null count
+  return Status::OK();
+}
+
+// cond.valid && (cond.data && left.valid || ~cond.data && right.valid)
+// ASS
+Status PromoteNulls(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                    const Scalar& right, ArrayData* output) {
+  if (!cond.MayHaveNulls() && left.is_valid && right.is_valid) {
+    return Status::OK();  // no nulls to handle
+  }
+  const int64_t len = cond.length;
+
+  std::shared_ptr<Buffer> out_validity;
+  if (right.is_valid) {
+    // out_validity = ~cond.data
+    ARROW_ASSIGN_OR_RAISE(
+        out_validity, arrow::internal::InvertBitmap(
+                          ctx->memory_pool(), cond.buffers[1]->data(), cond.offset, len));
+  } else {
+    // out_validity = [0...]
+    ARROW_ASSIGN_OR_RAISE(out_validity, ctx->AllocateBitmap(len));
+  }
+
+  // out_validity = cond.data && left.valid || ~cond.data && right.valid
+  if (left.is_valid) {
+    arrow::internal::BitmapOr(out_validity->data(), 0, cond.buffers[1]->data(),
+                              cond.offset, len, 0, out_validity->mutable_data());
+  }
+
+  // out_validity = cond.valid && (cond.data && left.valid || ~cond.data && right.valid)
+  if (cond.MayHaveNulls()) {
+    ::arrow::internal::BitmapAnd(out_validity->data(), 0, cond.buffers[0]->data(),
+                                 cond.offset, len, 0, out_validity->mutable_data());
+  }
+
+  output->buffers[0] = std::move(out_validity);
+  output->GetNullCount();  // update null count
+  return Status::OK();
+}
+
+// todo: this could be dangerous because the inverted arraydata buffer[1] may not be
+//  available outside Exec's scope
+Status InvertBoolArrayData(KernelContext* ctx, const ArrayData& input,
+                           ArrayData* output) {
+  // null buffer
+  if (input.MayHaveNulls()) {
+    output->buffers.emplace_back(
+        SliceBuffer(input.buffers[0], input.offset, input.length));
+  } else {
+    output->buffers.push_back(NULLPTR);
+  }
+
+  // data buffer
+  ARROW_ASSIGN_OR_RAISE(
+      std::shared_ptr<Buffer> inv_data,
+      arrow::internal::InvertBitmap(ctx->memory_pool(), input.buffers[1]->data(),
+                                    input.offset, input.length));
+  output->buffers.emplace_back(std::move(inv_data));
+  return Status::OK();
+}
+ */
+
+template <typename Type, typename Enable = void>
+struct IfElseFunctor {};
+
+// only number types needs to be handled for Fixed sized primitive data types because,
+// internal::GenerateTypeAgnosticPrimitive forwards types to the corresponding unsigned
+// int type
+template <typename Type>
+struct IfElseFunctor<Type, enable_if_number<Type>> {
+  using T = typename TypeTraits<Type>::CType;
+  // A - Array
+  // S - Scalar
+
+  //  AAA
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                     const ArrayData& right, ArrayData* out) {
+    ARROW_RETURN_NOT_OK(PromoteNullsVisitor(ctx, cond, left, right, out));
+
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> out_buf,
+                          ctx->Allocate(cond.length * sizeof(T)));
+    T* out_values = reinterpret_cast<T*>(out_buf->mutable_data());
+
+    // copy right data to out_buff
+    const T* right_data = right.GetValues<T>(1);
+    std::memcpy(out_values, right_data, right.length * sizeof(T));
+
+    const auto* cond_data = cond.buffers[1]->data();  // this is a BoolArray
+    BitBlockCounter bit_counter(cond_data, cond.offset, cond.length);
+
+    // selectively copy values from left data
+    const T* left_data = left.GetValues<T>(1);
+    int64_t offset = cond.offset;
+
+    // todo this can be improved by intrinsics. ex: _mm*_mask_store_e* (vmovdqa*)
+    while (offset < cond.offset + cond.length) {
+      const BitBlockCount& block = bit_counter.NextWord();
+      if (block.AllSet()) {  // all from left
+        std::memcpy(out_values, left_data, block.length * sizeof(T));
+      } else if (block.popcount) {  // selectively copy from left
+        for (int64_t i = 0; i < block.length; ++i) {
+          if (BitUtil::GetBit(cond_data, offset + i)) {
+            out_values[i] = left_data[i];
+          }
+        }
+      }
+
+      offset += block.length;
+      out_values += block.length;
+      left_data += block.length;
+    }
+
+    out->buffers[1] = std::move(out_buf);
+    return Status::OK();
+  }
+
+  // ASA
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                     const ArrayData& right, ArrayData* out) {
+    ARROW_RETURN_NOT_OK(PromoteNullsVisitor(ctx, cond, left, right, out));
+
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> out_buf,
+                          ctx->Allocate(cond.length * sizeof(T)));
+    T* out_values = reinterpret_cast<T*>(out_buf->mutable_data());
+
+    // copy right data to out_buff
+    const T* right_data = right.GetValues<T>(1);
+    std::memcpy(out_values, right_data, right.length * sizeof(T));
+
+    const auto* cond_data = cond.buffers[1]->data();  // this is a BoolArray
+    BitBlockCounter bit_counter(cond_data, cond.offset, cond.length);
+
+    // selectively copy values from left data
+    T left_data = internal::UnboxScalar<Type>::Unbox(left);
+    int64_t offset = cond.offset;
+
+    // todo this can be improved by intrinsics. ex: _mm*_mask_store_e* (vmovdqa*)
+    while (offset < cond.offset + cond.length) {
+      const BitBlockCount& block = bit_counter.NextWord();
+      if (block.AllSet()) {  // all from left
+        std::fill(out_values, out_values + block.length, left_data);
+      } else if (block.popcount) {  // selectively copy from left
+        for (int64_t i = 0; i < block.length; ++i) {
+          if (BitUtil::GetBit(cond_data, offset + i)) {
+            out_values[i] = left_data;
+          }
+        }
+      }
+
+      offset += block.length;
+      out_values += block.length;
+    }
+
+    out->buffers[1] = std::move(out_buf);
+    return Status::OK();
+  }
+
+  // AAS
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                     const Scalar& right, ArrayData* out) {
+    ARROW_RETURN_NOT_OK(PromoteNullsVisitor(ctx, cond, left, right, out));
+
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> out_buf,
+                          ctx->Allocate(cond.length * sizeof(T)));
+    T* out_values = reinterpret_cast<T*>(out_buf->mutable_data());
+
+    // copy left data to out_buff
+    const T* left_data = left.GetValues<T>(1);
+    std::memcpy(out_values, left_data, left.length * sizeof(T));
+
+    const auto* cond_data = cond.buffers[1]->data();  // this is a BoolArray
+    BitBlockCounter bit_counter(cond_data, cond.offset, cond.length);
+
+    // selectively copy values from left data
+    T right_data = internal::UnboxScalar<Type>::Unbox(right);
+    int64_t offset = cond.offset;
+
+    // todo this can be improved by intrinsics. ex: _mm*_mask_store_e* (vmovdqa*)
+    // left data is already in the output buffer. Therefore, mask needs to be inverted
+    while (offset < cond.offset + cond.length) {
+      const BitBlockCount& block = bit_counter.NextWord();
+      if (block.NoneSet()) {  // all from right
+        std::fill(out_values, out_values + block.length, right_data);
+      } else if (block.popcount) {  // selectively copy from right
+        for (int64_t i = 0; i < block.length; ++i) {
+          if (!BitUtil::GetBit(cond_data, offset + i)) {
+            out_values[i] = right_data;
+          }
+        }
+      }
+
+      offset += block.length;
+      out_values += block.length;
+    }
+
+    out->buffers[1] = std::move(out_buf);
+    return Status::OK();
+  }
+
+  // ASS
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                     const Scalar& right, ArrayData* out) {
+    ARROW_RETURN_NOT_OK(PromoteNullsVisitor(ctx, cond, left, right, out));
+
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> out_buf,
+                          ctx->Allocate(cond.length * sizeof(T)));
+    T* out_values = reinterpret_cast<T*>(out_buf->mutable_data());
+
+    // copy right data to out_buff
+    T right_data = internal::UnboxScalar<Type>::Unbox(right);
+    std::fill(out_values, out_values + cond.length, right_data);
+
+    const auto* cond_data = cond.buffers[1]->data();  // this is a BoolArray
+    BitBlockCounter bit_counter(cond_data, cond.offset, cond.length);
+
+    // selectively copy values from left data
+    T left_data = internal::UnboxScalar<Type>::Unbox(left);
+    int64_t offset = cond.offset;
+
+    // todo this can be improved by intrinsics. ex: _mm*_mask_store_e* (vmovdqa*)
+    while (offset < cond.offset + cond.length) {
+      const BitBlockCount& block = bit_counter.NextWord();
+      if (block.AllSet()) {  // all from left
+        std::fill(out_values, out_values + block.length, left_data);
+      } else if (block.popcount) {  // selectively copy from left
+        for (int64_t i = 0; i < block.length; ++i) {
+          if (BitUtil::GetBit(cond_data, offset + i)) {
+            out_values[i] = left_data;
+          }
+        }
+      }
+
+      offset += block.length;
+      out_values += block.length;
+    }
+
+    out->buffers[1] = std::move(out_buf);
+    return Status::OK();
+  }
+};
+
+template <typename Type>
+struct IfElseFunctor<Type, enable_if_boolean<Type>> {
+  // AAA
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                     const ArrayData& right, ArrayData* out) {
+    ARROW_RETURN_NOT_OK(PromoteNullsVisitor(ctx, cond, left, right, out));
+
+    // out_buff = right & ~cond
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> out_buf,
+                          arrow::internal::BitmapAndNot(
+                              ctx->memory_pool(), right.buffers[1]->data(), right.offset,
+                              cond.buffers[1]->data(), cond.offset, cond.length, 0));
+
+    // out_buff = left & cond
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> temp_buf,
+                          arrow::internal::BitmapAnd(
+                              ctx->memory_pool(), left.buffers[1]->data(), left.offset,
+                              cond.buffers[1]->data(), cond.offset, cond.length, 0));
+
+    arrow::internal::BitmapOr(out_buf->data(), 0, temp_buf->data(), 0, cond.length, 0,
+                              out_buf->mutable_data());
+    out->buffers[1] = std::move(out_buf);
+    return Status::OK();
+  }
+
+  // ASA
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                     const ArrayData& right, ArrayData* out) {
+    ARROW_RETURN_NOT_OK(PromoteNullsVisitor(ctx, cond, left, right, out));
+
+    // out_buff = right & ~cond
+    ARROW_ASSIGN_OR_RAISE(std::shared_ptr<Buffer> out_buf,
+                          arrow::internal::BitmapAndNot(
+                              ctx->memory_pool(), right.buffers[1]->data(), right.offset,
+                              cond.buffers[1]->data(), cond.offset, cond.length, 0));
+
+    // out_buff = left & cond
+    bool left_data = internal::UnboxScalar<BooleanType>::Unbox(left);
+    if (left_data) {
+      arrow::internal::BitmapOr(out_buf->data(), 0, cond.buffers[1]->data(), cond.offset,
+                                cond.length, 0, out_buf->mutable_data());
+    }
+
+    out->buffers[1] = std::move(out_buf);
+    return Status::OK();
+  }
+
+  // AAS
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                     const Scalar& right, ArrayData* out) {
+    // todo impl
+    return Status::OK();
+  }
+
+  // ASS
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                     const Scalar& right, ArrayData* out) {
+    // todo impl
+    return Status::OK();
+  }
+};
+
+template <typename Type>
+struct IfElseFunctor<Type, enable_if_null<Type>> {
+  template <typename T>
+  static inline Status ReturnCopy(const T& in, T* out) {
+    // Nothing preallocated, so we assign in into the output
+    *out = in;
+    return Status::OK();
+  }
+
+  // AAA
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                     const ArrayData& right, ArrayData* out) {
+    return ReturnCopy(left, out);
+  }
+
+  // ASA
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                     const ArrayData& right, ArrayData* out) {
+    return ReturnCopy(right, out);
+  }
+
+  // AAS
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const ArrayData& left,
+                     const Scalar& right, ArrayData* out) {
+    return ReturnCopy(left, out);
+  }
+
+  // ASS
+  static Status Call(KernelContext* ctx, const ArrayData& cond, const Scalar& left,
+                     const Scalar& right, ArrayData* out) {
+    return ReturnCopy(cond, out);
+  }
+};
+
+template <typename Type>
+struct ResolveIfElseExec {
+  static Status Exec(KernelContext* ctx, const ExecBatch& batch, Datum* out) {
+    // cond is scalar

Review comment:
       Scalar conditions don't seem tested below?




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org